DagsterDocs

Solids

The foundational unit of composition in Dagster.


Defining solids

@dagster.solid(name=None, description=None, input_defs=None, output_defs=None, config_schema=None, required_resource_keys=None, tags=None, version=None)[source]

Create a solid with the specified parameters from the decorated function.

This shortcut simplifies the core SolidDefinition API by exploding arguments into kwargs of the decorated compute function and omitting additional parameters when they are not needed.

Input and output definitions will be inferred from the type signature of the decorated function if not explicitly provided.

The decorated function will be used as the solid’s compute function. The signature of the decorated function is more flexible than that of the compute_fn in the core API; it may:

  1. Return a value. This value will be wrapped in an Output and yielded by the compute function.

  2. Return an Output. This output will be yielded by the compute function.

  3. Yield Output or other event objects. Same as default compute behavior.

Note that options 1) and 2) are incompatible with yielding other events – if you would like to decorate a function that yields events, it must also wrap its eventual output in an Output and yield it.

@solid supports async def functions as well, including async generators when yielding multiple events or outputs. Note that async solids will generally be run on their own unless using a custom Executor implementation that supports running them together.

Parameters
  • name (Optional[str]) – Name of solid. Must be unique within any PipelineDefinition using the solid.

  • description (Optional[str]) – Human-readable description of this solid. If not provided, and the decorated function has docstring, that docstring will be used as the description.

  • input_defs (Optional[List[InputDefinition]]) – Information about the inputs to the solid. Information provided here will be combined with what can be inferred from the function signature, with these explicit InputDefinitions taking precedence.

  • output_defs (Optional[List[OutputDefinition]]) – Information about the solids outputs. Information provided here will be combined with what can be inferred from the return type signature if there is only one OutputDefinition and the function does not use yield.

  • config_schema (Optional[ConfigSchema) – The schema for the config. If set, Dagster will check that config provided for the solid matches this schema and fail if it does not. If not set, Dagster will accept any config provided for the solid.

  • required_resource_keys (Optional[Set[str]]) – Set of resource handles required by this solid.

  • tags (Optional[Dict[str, Any]]) – Arbitrary metadata for the solid. Frameworks may expect and require certain metadata to be attached to a solid. Users should generally not set metadata directly. Values that are not strings will be json encoded and must meet the criteria that json.loads(json.dumps(value)) == value.

  • version (Optional[str]) – (Experimental) The version of the solid’s compute_fn. Two solids should have the same version if and only if they deterministically produce the same outputs when provided the same inputs.

Examples

@solid
def hello_world(_context):
    print('hello')

@solid
def hello_world(_context):
    return {'foo': 'bar'}

@solid
def hello_world(_context):
    return Output(value={'foo': 'bar'})

@solid
def hello_world(_context):
    yield Output(value={'foo': 'bar'})

@solid
def hello_world(_context, foo):
    return foo

@solid(
    input_defs=[InputDefinition(name="foo", str)],
    output_defs=[OutputDefinition(str)]
)
def hello_world(_context, foo):
    # explicitly type and name inputs and outputs
    return foo

@solid
def hello_world(_context, foo: str) -> str:
    # same as above inferred from signature
    return foo

@solid
def hello_world(context, foo):
    context.log.info('log something')
    return foo

@solid(
    config_schema={'str_value' : Field(str)}
)
def hello_world(context, foo):
    # context.solid_config is a dictionary with 'str_value' key
    return foo + context.solid_config['str_value']
class dagster.SolidDefinition(name, input_defs, compute_fn, output_defs, config_schema=None, description=None, tags=None, required_resource_keys=None, positional_inputs=None, version=None)[source]

The definition of a Solid that performs a user-defined computation.

For more details on what a solid is, refer to the Solid Overview .

End users should prefer the @solid and @lambda_solid decorators. SolidDefinition is generally intended to be used by framework authors.

Parameters
  • name (str) – Name of the solid. Must be unique within any PipelineDefinition using the solid.

  • input_defs (List[InputDefinition]) – Inputs of the solid.

  • compute_fn (Callable) –

    The core of the solid, the function that does the actual computation. The signature of this function is determined by input_defs, with an additional injected first argument, context, a collection of information provided by the system.

    This function must return a generator or an async generator, which must yield one Output for each of the solid’s output_defs, and additionally may yield other types of Dagster events, including Materialization and ExpectationResult.

  • output_defs (List[OutputDefinition]) – Outputs of the solid.

  • config_schema (Optional[ConfigSchema) – The schema for the config. If set, Dagster will check that config provided for the solid matches this schema and fail if it does not. If not set, Dagster will accept any config provided for the solid.

  • description (Optional[str]) – Human-readable description of the solid.

  • tags (Optional[Dict[str, Any]]) – Arbitrary metadata for the solid. Frameworks may expect and require certain metadata to be attached to a solid. Users should generally not set metadata directly. Values that are not strings will be json encoded and must meet the criteria that json.loads(json.dumps(value)) == value.

  • required_resource_keys (Optional[Set[str]]) – Set of resources handles required by this solid.

  • positional_inputs (Optional[List[str]]) – The positional order of the input names if it differs from the order of the input definitions.

  • version (Optional[str]) – (Experimental) The version of the solid’s compute_fn. Two solids should have the same version if and only if they deterministically produce the same outputs when provided the same inputs.

Examples

def _add_one(_context, inputs):
    yield Output(inputs["num"] + 1)

SolidDefinition(
    name="add_one",
    input_defs=[InputDefinition("num", Int)],
    output_defs=[OutputDefinition(Int)], # default name ("result")
    compute_fn=_add_one,
)
configured(config_or_config_fn, name, config_schema=None, description=None)

Wraps this object in an object of the same type that provides configuration to the inner object.

Parameters
  • config_or_config_fn (Union[Any, Callable[[Any], Any]]) – Either (1) Run configuration that fully satisfies this object’s config schema or (2) A function that accepts run configuration and returns run configuration that fully satisfies this object’s config schema. In the latter case, config_schema must be specified. When passing a function, it’s easiest to use configured().

  • name (str) – Name of the new definition. This is a required argument, as this definition type has a name uniqueness constraint.

  • config_schema (ConfigSchema) – If config_or_config_fn is a function, the config schema that its input must satisfy.

  • description (Optional[str]) – Description of the new definition. If not specified, inherits the description of the definition being configured.

Returns (ConfigurableDefinition): A configured version of this object.

@dagster.lambda_solid(name=None, description=None, input_defs=None, output_def=None)[source]

Create a simple solid from the decorated function.

This shortcut allows the creation of simple solids that do not require configuration and whose implementations do not require a context.

Lambda solids take any number of inputs and produce a single output.

Inputs can be defined using InputDefinition and passed to the input_defs argument of this decorator, or inferred from the type signature of the decorated function.

The single output can be defined using OutputDefinition and passed as the output_def argument of this decorator, or its type can be inferred from the type signature of the decorated function.

The body of the decorated function should return a single value, which will be yielded as the solid’s output.

Parameters

Examples:

@lambda_solid
def hello_world():
    return 'hello'

@lambda_solid(
    input_defs=[InputDefinition(name='foo', str)],
    output_def=OutputDefinition(str)
)
def hello_world(foo):
    # explicitly type and name inputs and outputs
    return foo

@lambda_solid
def hello_world(foo: str) -> str:
    # same as above inferred from signature
    return foo

Inputs & outputs

class dagster.InputDefinition(name, dagster_type=None, description=None, default_value=<class 'dagster.core.definitions.utils.NoValueSentinel'>, root_manager_key=None, metadata=None, asset_key=None, asset_partitions=None)[source]

Defines an argument to a solid’s compute function.

Inputs may flow from previous solids’ outputs, or be stubbed using config. They may optionally be typed using the Dagster type system.

Parameters
  • name (str) – Name of the input.

  • dagster_type (Optional[Union[Type, DagsterType]]]) – The type of this input. Users should provide the Python type of the objects that they expect to be passed for this input, or a DagsterType that defines a runtime check that they want to be run on this input. Defaults to Any.

  • description (Optional[str]) – Human-readable description of the input.

  • default_value (Optional[Any]) – The default value to use if no input is provided.

  • root_manager_key (Optional[str]) – (Experimental) The resource key for the RootInputManager used for loading this input when it is not connected to an upstream output.

  • metadata (Optional[Dict[str, Any]]) – (Experimental) A dict of metadata for the input.

  • asset_key (Optional[Union[AssetKey, InputContext -> AssetKey]]) – (Experimental) An AssetKey (or function that produces an AssetKey from the InputContext) which should be associated with this InputDefinition. Used for tracking lineage information through Dagster.

  • asset_partitions (Optional[Union[Set[str], InputContext -> Set[str]]]) – (Experimental) A set of partitions of the given asset_key (or a function that produces this list of partitions from the InputContext) which should be associated with this InputDefinition.

class dagster.OutputDefinition(dagster_type=None, name=None, description=None, is_required=None, io_manager_key=None, metadata=None, asset_key=None, asset_partitions=None)[source]

Defines an output from a solid’s compute function.

Solids can have multiple outputs, in which case outputs cannot be anonymous.

Many solids have only one output, in which case the user can provide a single output definition that will be given the default name, “result”.

Output definitions may be typed using the Dagster type system.

Parameters
  • dagster_type (Optional[Union[Type, DagsterType]]]) – The type of this output. Users should provide the Python type of the objects that they expect the solid to yield for this output, or a DagsterType that defines a runtime check that they want to be run on this output. Defaults to Any.

  • name (Optional[str]) – Name of the output. (default: “result”)

  • description (Optional[str]) – Human-readable description of the output.

  • is_required (Optional[bool]) – Whether the presence of this field is required. (default: True)

  • io_manager_key (Optional[str]) – The resource key of the output manager used for this output. (default: “io_manager”).

  • metadata (Optional[Dict[str, Any]]) – (Experimental) A dict of the metadata for the output. For example, users can provide a file path if the data object will be stored in a filesystem, or provide information of a database table when it is going to load the data into the table.

  • asset_key (Optional[Union[AssetKey, OutputContext -> AssetKey]]) – (Experimental) An AssetKey (or function that produces an AssetKey from the OutputContext) which should be associated with this OutputDefinition. Used for tracking lineage information through Dagster.

  • asset_partitions (Optional[Union[Set[str], OutputContext -> Set[str]]]) – (Experimental) A set of partitions of the given asset_key (or a function that produces this list of partitions from the OutputContext) which should be associated with this OutputDefinition.


Composing solids

@dagster.composite_solid(name=None, input_defs=None, output_defs=None, description=None, config_schema=None, config_fn=None)[source]

Create a composite solid with the specified parameters from the decorated composition function.

Using this decorator allows you to build up the dependency graph of the composite by writing a function that invokes solids and passes the output to other solids. This is similar to the use of the @pipeline decorator, with the additional ability to remap inputs, outputs, and config across the composite boundary.

Parameters
  • name (Optional[str]) – Name for the new composite solid. Must be unique within any PipelineDefinition using the solid.

  • description (Optional[str]) – Human-readable description of the new composite solid.

  • input_defs (Optional[List[InputDefinition]]) –

    Information about the inputs that this composite solid maps. Information provided here will be combined with what can be inferred from the function signature, with these explicit InputDefinitions taking precedence.

    Uses of inputs in the body of the decorated composition function will determine the InputMappings passed to the underlying CompositeSolidDefinition.

  • output_defs (Optional[List[OutputDefinition]]) –

    Information about the outputs this composite solid maps. Information provided here will be combined with what can be inferred from the return type signature if there is only one OutputDefinition.

    Uses of these outputs in the body of the decorated composition function, as well as the return value of the decorated function, will be used to infer the appropriate set of OutputMappings for the underlying CompositeSolidDefinition.

    To map multiple outputs, return a dictionary from the composition function.

  • config_schema (Optional[ConfigSchema]) – If the config_fn argument is provided, this argument can be provided to set the schema for outer config that is passed to the config_fn. If config_fn is provided, but this argument is not provided, any config will be accepted.

  • config_fn (Callable[[dict], dict]) –

    By specifying a config mapping function, you can override the configuration for the child solids contained within this composite solid. config_fn, maps the config provided to the composite solid to the config that will be provided to the child solids.

    If this argument is provided, the config_schema argument can also be provided to limit what config values can be passed to the composite solid.

Examples

@lambda_solid
def add_one(num: int) -> int:
    return num + 1

@composite_solid
def add_two(num: int) -> int:
    adder_1 = add_one.alias('adder_1')
    adder_2 = add_one.alias('adder_2')

    return adder_2(adder_1(num))
class dagster.CompositeSolidDefinition(name, solid_defs, input_mappings=None, output_mappings=None, config_mapping=None, dependencies=None, description=None, tags=None, positional_inputs=None)[source]

The core unit of composition and abstraction, composite solids allow you to define a solid from a graph of solids.

In the same way you would refactor a block of code in to a function to deduplicate, organize, or manage complexity - you can refactor solids in a pipeline in to a composite solid.

Parameters
  • name (str) – The name of this composite solid. Must be unique within any PipelineDefinition using the solid.

  • solid_defs (List[Union[SolidDefinition, CompositeSolidDefinition]]) – The set of solid definitions used in this composite solid. Composites may be arbitrarily nested.

  • input_mappings (Optional[List[InputMapping]]) – Define the inputs to the composite solid, and how they map to the inputs of its constituent solids.

  • output_mappings (Optional[List[OutputMapping]]) – Define the outputs of the composite solid, and how they map from the outputs of its constituent solids.

  • config_mapping (Optional[ConfigMapping]) – By specifying a config mapping, you can override the configuration for the child solids contained within this composite solid. Config mappings require both a configuration field to be specified, which is exposed as the configuration for the composite solid, and a configuration mapping function, which is called to map the configuration of the composite solid into the configuration that is applied to any child solids.

  • dependencies (Optional[Dict[Union[str, SolidInvocation], Dict[str, DependencyDefinition]]]) – A structure that declares where each solid gets its inputs. The keys at the top level dict are either string names of solids or SolidInvocations. The values are dicts that map input names to DependencyDefinitions.

  • description (Optional[str]) – Human readable description of this composite solid.

  • tags (Optional[Dict[str, Any]]) – Arbitrary metadata for the solid. Frameworks may expect and require certain metadata to be attached to a solid. Users should generally not set metadata directly. Values that are not strings will be json encoded and must meet the criteria that json.loads(json.dumps(value)) == value. may expect and require certain metadata to be attached to a solid.

  • positional_inputs (Optional[List[str]]) – The positional order of the inputs if it differs from the order of the input mappings

Examples

@lambda_solid
def add_one(num: int) -> int:
    return num + 1

add_two = CompositeSolidDefinition(
    'add_two',
    solid_defs=[add_one],
    dependencies={
        SolidInvocation('add_one', 'adder_1'): {},
        SolidInvocation('add_one', 'adder_2'): {'num': DependencyDefinition('adder_1')},
    },
    input_mappings=[InputDefinition('num', Int).mapping_to('adder_1', 'num')],
    output_mappings=[OutputDefinition(Int).mapping_from('adder_2')],
)
configured(config_or_config_fn, name, config_schema=None, description=None)

Wraps this object in an object of the same type that provides configuration to the inner object.

Parameters
  • config_or_config_fn (Union[Any, Callable[[Any], Any]]) – Either (1) Run configuration that fully satisfies this object’s config schema or (2) A function that accepts run configuration and returns run configuration that fully satisfies this object’s config schema. In the latter case, config_schema must be specified. When passing a function, it’s easiest to use configured().

  • name (str) – Name of the new definition. This is a required argument, as this definition type has a name uniqueness constraint.

  • config_schema (ConfigSchema) – If config_or_config_fn is a function, the config schema that its input must satisfy.

  • description (Optional[str]) – Description of the new definition. If not specified, inherits the description of the definition being configured.

Returns (ConfigurableDefinition): A configured version of this object.

class dagster.InputMapping(definition, maps_to)[source]

Defines an input mapping for a composite solid.

Parameters
  • definition (InputDefinition) – Defines the input to the composite solid.

  • solid_name (str) – The name of the child solid onto which to map the input.

  • input_name (str) – The name of the input to the child solid onto which to map the input.

class dagster.OutputMapping(definition, maps_from)[source]

Defines an output mapping for a composite solid.

Parameters
  • definition (OutputDefinition) – Defines the output of the composite solid.

  • solid_name (str) – The name of the child solid from which to map the output.

  • output_name (str) – The name of the child solid’s output from which to map the output.

class dagster.ConfigMapping(config_fn, config_schema=None)[source]

Defines a config mapping for a composite solid.

By specifying a config mapping function, you can override the configuration for the child solids contained within a composite solid.

Config mappings require the configuration schema to be specified as config_schema, which will be exposed as the configuration schema for the composite solid, as well as a configuration mapping function, config_fn, which maps the config provided to the composite solid to the config that will be provided to the child solids.

Parameters
  • config_fn (Callable[[dict], dict]) – The function that will be called to map the composite config to a config appropriate for the child solids.

  • config_schema (ConfigSchema) – The schema of the composite config.

Events

The objects that can be yielded by the body of solids’ compute functions to communicate with the Dagster framework.

(Note that Failure and RetryRequested are intended to be raised from solids rather than yielded.)

Event types

class dagster.Output(value, output_name='result', metadata_entries=None)[source]

Event corresponding to one of a solid’s outputs.

Solid compute functions must explicitly yield events of this type when they have more than one output, or when they also yield events of other types, or when defining a solid using the SolidDefinition API directly.

Outputs are values produced by solids that will be consumed by downstream solids in a pipeline. They are type-checked at solid boundaries when their corresponding OutputDefinition or the downstream InputDefinition is typed.

Parameters
  • value (Any) – The value returned by the compute function.

  • output_name (Optional[str]) – Name of the corresponding output definition. (default: “result”)

  • metadata_entries (Optional[Union[EventMetadataEntry, PartitionMetadataEntry]]) – (Experimental) A set of metadata entries to attach to events related to this Output.

class dagster.AssetMaterialization(asset_key, description=None, metadata_entries=None, partition=None, tags=None)[source]

Event indicating that a solid has materialized an asset.

Solid compute functions may yield events of this type whenever they wish to indicate to the Dagster framework (and the end user) that they have produced a materialized value as a side effect of computation. Unlike outputs, asset materializations can not be passed to other solids, and their persistence is controlled by solid logic, rather than by the Dagster framework.

Solid authors should use these events to organize metadata about the side effects of their computations, enabling tooling like the Assets dashboard in Dagit.

Parameters
  • asset_key (str|List[str]|AssetKey) – A key to identify the materialized asset across pipeline runs

  • description (Optional[str]) – A longer human-readable description of the materialized value.

  • metadata_entries (Optional[List[EventMetadataEntry]]) – Arbitrary metadata about the materialized value.

  • partition (Optional[str]) – The name of the partition that was materialized.

  • tags (Dict[str, str]) – (Experimental) Metadata for a given asset materialization. Used for search and organization of the asset entry in the asset catalog in Dagit.

static file(path, description=None, asset_key=None)[source]

Static constructor for standard materializations corresponding to files on disk.

Parameters
  • path (str) – The path to the file.

  • description (Optional[str]) – A human-readable description of the materialization.

class dagster.ExpectationResult(success, label=None, description=None, metadata_entries=None)[source]

Event corresponding to a data quality test.

Solid compute functions may yield events of this type whenever they wish to indicate to the Dagster framework (and the end user) that a data quality test has produced a (positive or negative) result.

Parameters
  • success (bool) – Whether the expectation passed or not.

  • label (Optional[str]) – Short display name for expectation. Defaults to “result”.

  • description (Optional[str]) – A longer human-readable description of the expectation.

  • metadata_entries (Optional[List[EventMetadataEntry]]) – Arbitrary metadata about the expectation.

class dagster.TypeCheck(success, description=None, metadata_entries=None)[source]

Event corresponding to a successful typecheck.

Events of this type should be returned by user-defined type checks when they need to encapsulate additional metadata about a type check’s success or failure. (i.e., when using as_dagster_type(), @usable_as_dagster_type, or the underlying PythonObjectDagsterType() API.)

Solid compute functions should generally avoid yielding events of this type to avoid confusion.

Parameters
  • success (bool) – True if the type check succeeded, False otherwise.

  • description (Optional[str]) – A human-readable description of the type check.

  • metadata_entries (Optional[List[EventMetadataEntry]]) – Arbitrary metadata about the type check.

class dagster.Failure(description=None, metadata_entries=None)[source]

Event indicating solid failure.

Raise events of this type from within solid compute functions or custom type checks in order to indicate an unrecoverable failure in user code to the Dagster machinery and return structured metadata about the failure.

Parameters
  • description (Optional[str]) – A human-readable description of the failure.

  • metadata_entries (Optional[List[EventMetadataEntry]]) – Arbitrary metadata about the failure.

class dagster.RetryRequested(max_retries=1, seconds_to_wait=None)[source]

An exception to raise from a solid to indicate that it should be retried.

Parameters
  • max_retries (Optional[int]) – The max number of retries this step should attempt before failing

  • seconds_to_wait (Optional[int]) – Seconds to wait before restarting the step after putting the step in to the up_for_retry state

Example

@solid
def flakes():
    try:
        flakey_operation()
    except:
        raise RetryRequested(max_retries=3)

Metadata entries

Dagster uses lists of metadata entries to communicate arbitrary user-specified metadata about structured events.

class dagster.EventMetadataEntry(label, description, entry_data)[source]

The standard structure for describing metadata for Dagster events.

Lists of objects of this type can be passed as arguments to Dagster events and will be displayed in Dagit and other tooling.

Parameters
static float(value, label, description=None)[source]

Static constructor for a metadata entry containing float as FloatMetadataEntryData. For example:

@solid
def emit_metadata_solid(context, df):
    yield AssetMaterialization(
        asset_key="my_dataset",
        metadata_entries=[EventMetadataEntry.float(calculate_bytes(df), "size (bytes)")],
    )
Parameters
  • value (Optional[float]) – The float value contained by this metadata entry.

  • label (str) – Short display label for this metadata entry.

  • description (Optional[str]) – A human-readable description of this metadata entry.

static fspath(path, label=None, description=None)[source]

Static constructor for a metadata entry containing a filesystem path as PathMetadataEntryData. For example:

@solid
def emit_metadata_solid(context):
    yield AssetMaterialization(
        asset_key="my_dataset",
        metadata_entries=[EventMetadataEntry.fspath("path/to/file")],
    )
Parameters
  • path (Optional[str]) – The path contained by this metadata entry.

  • label (str) – Short display label for this metadata entry. Defaults to the base name of the path.

  • description (Optional[str]) – A human-readable description of this metadata entry.

static int(value, label, description=None)[source]

Static constructor for a metadata entry containing int as IntMetadataEntryData. For example:

@solid
def emit_metadata_solid(context, df):
    yield AssetMaterialization(
        asset_key="my_dataset",
        metadata_entries=[EventMetadataEntry.int(len(df), "number of rows")],
    )
Parameters
  • value (Optional[int]) – The int value contained by this metadata entry.

  • label (str) – Short display label for this metadata entry.

  • description (Optional[str]) – A human-readable description of this metadata entry.

static json(data, label, description=None)[source]

Static constructor for a metadata entry containing JSON data as JsonMetadataEntryData. For example:

@solid
def emit_metadata_solid(context):
    yield ExpectationResult(
        success=not missing_things,
        label="is_present",
        metadata_entries=[
            EventMetadataEntry.json(
                label="metadata", data={"missing_columns": missing_things},
            )
        ],
    )
Parameters
  • data (Optional[Dict[str, Any]]) – The JSON data contained by this metadata entry.

  • label (str) – Short display label for this metadata entry.

  • description (Optional[str]) – A human-readable description of this metadata entry.

static md(md_str, label, description=None)[source]

Static constructor for a metadata entry containing markdown data as MarkdownMetadataEntryData. For example:

@solid
def emit_metadata_solid(context, md_str):
    yield AssetMaterialization(
        asset_key="info",
        metadata_entries=[EventMetadataEntry.md(md_str=md_str)],
    )
Parameters
  • md_str (Optional[str]) – The markdown contained by this metadata entry.

  • label (str) – Short display label for this metadata entry.

  • description (Optional[str]) – A human-readable description of this metadata entry.

static path(path, label, description=None)[source]

Static constructor for a metadata entry containing a path as PathMetadataEntryData. For example:

@solid
def emit_metadata_solid(context):
    yield AssetMaterialization(
        asset_key="my_dataset",
        metadata_entries=[EventMetadataEntry.path("path/to/file", label="filepath")],
    )
Parameters
  • path (Optional[str]) – The path contained by this metadata entry.

  • label (str) – Short display label for this metadata entry.

  • description (Optional[str]) – A human-readable description of this metadata entry.

static text(text, label, description=None)[source]

Static constructor for a metadata entry containing text as TextMetadataEntryData. For example:

@solid
def emit_metadata_solid(context, df):
    yield AssetMaterialization(
        asset_key="my_dataset",
        metadata_entries=[
            EventMetadataEntry.text("Text-based metadata for this event", "text_metadata")
        ],
    )
Parameters
  • text (Optional[str]) – The text of this metadata entry.

  • label (str) – Short display label for this metadata entry.

  • description (Optional[str]) – A human-readable description of this metadata entry.

static url(url, label, description=None)[source]

Static constructor for a metadata entry containing a URL as UrlMetadataEntryData. For example:

@solid
def emit_metadata_solid(context):
    yield AssetMaterialization(
        asset_key="my_dashboard",
        metadata_entries=[
            EventMetadataEntry.url(
                "http://mycoolsite.com/my_dashboard", label="dashboard_url"
            ),
        ],
    )
Parameters
  • url (Optional[str]) – The URL contained by this metadata entry.

  • label (str) – Short display label for this metadata entry.

  • description (Optional[str]) – A human-readable description of this metadata entry.

class dagster.JsonMetadataEntryData(data)[source]

Container class for JSON metadata entry data.

Parameters

data (Optional[Dict[str, Any]]) – The JSON data.

class dagster.MarkdownMetadataEntryData(md_str)[source]

Container class for markdown metadata entry data.

Parameters

md_str (Optional[str]) – The markdown as a string.

class dagster.PathMetadataEntryData(path)[source]

Container class for path metadata entry data.

Parameters

path (Optional[str]) – The path as a string.

class dagster.TextMetadataEntryData(text)[source]

Container class for text metadata entry data.

Parameters

text (Optional[str]) – The text data.

class dagster.UrlMetadataEntryData(url)[source]

Container class for URL metadata entry data.

Parameters

url (Optional[str]) – The URL as a string.

class dagster.FloatMetadataEntryData(value)[source]

Container class for float metadata entry data.

Parameters

value (Optional[float]) – The float value.

class dagster.IntMetadataEntryData(value)[source]

Container class for int metadata entry data.

Parameters

value (Optional[int]) – The int value.


Asset Key

Dagster uses AssetKey to build an index on Materialization events. Assets materialized with an AssetKey are highlighted in dagit on the Assets dashboard.

class dagster.AssetKey(path=None)[source]

Object representing the structure of an asset key. Takes in a sanitized string, list of strings, or tuple of strings.

Example usage:

@solid
def emit_metadata_solid(context, df):
    yield AssetMaterialization(
        asset_key=AssetKey('flat_asset_key'),
        metadata_entries=[
            EventMetadataEntry.text("Text-based metadata for this event", "text_metadata")
        ],
    )

@solid
def structured_asset_key_solid(context, df):
    yield AssetMaterialization(
        asset_key=AssetKey(['parent', 'child', 'grandchild']),
        metadata_entries=[
            EventMetadataEntry.text("Text-based metadata for this event", "text_metadata")
        ],
    )

@solid
def structured_asset_key_solid_2(context, df):
    yield AssetMaterialization(
        asset_key=AssetKey(('parent', 'child', 'grandchild')),
        metadata_entries=[
            EventMetadataEntry.text("Text-based metadata for this event", "text_metadata")
        ],
    )
Parameters

path (str|str[]|str()) – String, list of strings, or tuple of strings. A list of strings represent the hierarchical structure of the asset_key.