Schedules and sensors
Dagster offers several ways to run data pipelines without manual intervention, including traditional scheduling and event-based triggers. Automating your Dagster pipelines can boost efficiency and ensure that data is produced consistently and reliably.
Run requests
- classdagster.RunRequest [source]
- Represents all the information required to launch a single run. Must be returned by a SensorDefinition or ScheduleDefinition’s evaluation function for a run to be launched. - Parameters: - run_key (Optional[str]) – A string key to identify this launched run. For sensors, ensures that only one run is created per run key across all sensor evaluations. For schedules, ensures that one run is created per tick, across failure recoveries. Passing in a None value means that a run will always be launched per evaluation.
- (Optional[Union[RunConfig (run_config) – Configuration for the run. If the job has a PartitionedConfig, this value will override replace the config provided by it.
- Mapping[str – Configuration for the run. If the job has a PartitionedConfig, this value will override replace the config provided by it.
- Any]]] – Configuration for the run. If the job has a PartitionedConfig, this value will override replace the config provided by it.
- tags (Optional[Dict[str, Any]]) – A dictionary of tags (string key-value pairs) to attach to the launched run.
- job_name (Optional[str]) – The name of the job this run request will launch. Required for sensors that target multiple jobs.
- asset_selection (Optional[Sequence[AssetKey]]) – A subselection of assets that should be launched with this run. If the sensor or schedule targets a job, then by default a RunRequest returned from it will launch all of the assets in the job. If the sensor targets an asset selection, then by default a RunRequest returned from it will launch all the assets in the selection. This argument is used to specify that only a subset of these assets should be launched, instead of all of them.
- asset_check_keys (Optional[Sequence[AssetCheckKey]]) – A subselection of asset checks that should be launched with this run. If the sensor/schedule targets a job, then by default a RunRequest returned from it will launch all of the asset checks in the job. If the sensor/schedule targets an asset selection, then by default a RunRequest returned from it will launch all the asset checks in the selection. This argument is used to specify that only a subset of these asset checks should be launched, instead of all of them.
- stale_assets_only (bool) – Set to true to further narrow the asset selection to stale assets. If passed without an asset selection, all stale assets in the job will be materialized. If the job does not materialize assets, this flag is ignored.
- partition_key (Optional[str]) – The partition key for this run request.
 
- classdagster.SkipReason [source]
- Represents a skipped evaluation, where no runs are requested. May contain a message to indicate why no runs were requested. - Parameters: skip_message (Optional[str]) – A message displayed in the Dagster UI for why this evaluation resulted in no requested runs. 
Schedules
Schedules are Dagster’s way to support traditional ways of automation, such as specifying a job should run at Mondays at 9:00AM. Jobs triggered by schedules can contain a subset of assets or ops.
- @dagster.schedule [source]
- Creates a schedule following the provided cron schedule and requests runs for the provided job. - The decorated function takes in a - ScheduleEvaluationContextas its only argument, and does one of the following:- Return a RunRequestobject.
- Return a list of RunRequestobjects.
- Return a SkipReasonobject, providing a descriptive message of why no runs were requested.
- Return nothing (skipping without providing a reason)
- Return a run config dictionary.
- Yield a SkipReasonor yield one ore moreRunRequestobjects. Returns aScheduleDefinition.
 - Parameters: - cron_schedule (Union[str, Sequence[str]]) – A valid cron string or sequence of cron strings specifying when the schedule will run, e.g., 45 23 * * 6for a schedule that runs at 11:45 PM every Saturday. If a sequence is provided, then the schedule will run for the union of all execution times for the provided cron strings, e.g.,['45 23 * * 6', '30 9 * * 0']for a schedule that runs at 11:45 PM every Saturday and 9:30 AM every Sunday.
- name (Optional[str]) – The name of the schedule.
- tags (Optional[Mapping[str, str]]) – A set of key-value tags that annotate the schedule and can be used for searching and filtering in the UI.
- tags_fn (Optional[Callable[[ScheduleEvaluationContext], Optional[Dict[str, str]]]]) – A function that generates tags to attach to the schedule’s runs. Takes a ScheduleEvaluationContextand returns a dictionary of tags (string key-value pairs). Note: Eithertagsortags_fnmay be set, but not both.
- metadata (Optional[Mapping[str, Any]]) – A set of metadata entries that annotate the schedule. Values will be normalized to typed MetadataValue objects.
- should_execute (Optional[Callable[[ScheduleEvaluationContext], bool]]) – A function that runs at schedule execution time to determine whether a schedule should execute or skip. Takes a ScheduleEvaluationContextand returns a boolean (Trueif the schedule should execute). Defaults to a function that always returnsTrue.
- execution_timezone (Optional[str]) – Timezone in which the schedule should run. Supported strings for timezones are the ones provided by the IANA time zone database - e.g. "America/Los_Angeles".
- description (Optional[str]) – A human-readable description of the schedule.
- job (Optional[Union[GraphDefinition, JobDefinition, UnresolvedAssetJobDefinition]]) – The job that should execute when the schedule runs.
- default_status (DefaultScheduleStatus) – If set to RUNNING, the schedule will immediately be active when starting Dagster. The default status can be overridden from the Dagster UI or via the GraphQL API.
- required_resource_keys (Optional[Set[str]]) – The set of resource keys required by the schedule.
- target (Optional[Union[CoercibleToAssetSelection, AssetsDefinition, JobDefinition, UnresolvedAssetJobDefinition]]) – The target that the schedule will execute. It can take AssetSelectionobjects and anything coercible to it (e.g. str, Sequence[str], AssetKey, AssetsDefinition). It can also acceptJobDefinition(a function decorated with @job is an instance of JobDefinition) and UnresolvedAssetJobDefinition (the return value ofdefine_asset_job()) objects. This parameter will replace job and job_name.
- owners (Optional[Sequence[str]]) – beta A sequence of strings identifying the owners of the schedule.
 
- Return a 
- classdagster.ScheduleDefinition [source]
- Defines a schedule that targets a job. - Parameters: - 
name (Optional[str]) – The name of the schedule to create. Defaults to the job name plus _schedule.
- 
cron_schedule (Union[str, Sequence[str]]) – A valid cron string or sequence of cron strings specifying when the schedule will run, e.g., 45 23 * * 6for a schedule that runs at 11:45 PM every Saturday. If a sequence is provided, then the schedule will run for the union of all execution times for the provided cron strings, e.g.,['45 23 * * 6', '30 9 * * 0]for a schedule that runs at 11:45 PM every Saturday and 9:30 AM every Sunday.
- 
execution_fn (Callable[ScheduleEvaluationContext]) – The core evaluation function for the schedule, which is run at an interval to determine whether a run should be launched or not. Takes a ScheduleEvaluationContext.
- 
run_config (Optional[Union[RunConfig, Mapping]]) – The config that parameterizes this execution, as a dict. 
- 
run_config_fn (Optional[Callable[[ScheduleEvaluationContext], [Mapping]]]) – A function that takes a ScheduleEvaluationContextobject and returns the run configuration that parameterizes this execution, as a dict. Note: Only one of the following may be set: You may setrun_config,run_config_fn, orexecution_fn.
- 
tags (Optional[Mapping[str, str]]) – A set of key-value tags that annotate the schedule and can be used for searching and filtering in the UI. If no execution_fn is provided, then these will also be automatically attached to runs launched by the schedule. 
- 
tags_fn (Optional[Callable[[ScheduleEvaluationContext], Optional[Mapping[str, str]]]]) – A function that generates tags to attach to the schedule’s runs. Takes a ScheduleEvaluationContextand returns a dictionary of tags (string key-value pairs). Note: Only one of the following may be set:tags,tags_fn, orexecution_fn.
- 
should_execute (Optional[Callable[[ScheduleEvaluationContext], bool]]) – A function that runs at schedule execution time to determine whether a schedule should execute or skip. Takes a ScheduleEvaluationContextand returns a boolean (Trueif the schedule should execute). Defaults to a function that always returnsTrue.
- 
execution_timezone (Optional[str]) – 
- 
description (Optional[str]) – A human-readable description of the schedule. 
- 
job (Optional[Union[GraphDefinition, JobDefinition]]) – The job that should execute when this schedule runs. 
- 
default_status (DefaultScheduleStatus) – If set to RUNNING, the schedule will start as running. The default status can be overridden from the Dagster UI or via the GraphQL API.
- 
required_resource_keys (Optional[Set[str]]) – The set of resource keys required by the schedule. 
- 
target (Optional[Union[CoercibleToAssetSelection, AssetsDefinition, JobDefinition, UnresolvedAssetJobDefinition]]) – The target that the schedule will execute. It can take AssetSelectionobjects and anything coercible to it (e.g. str, Sequence[str], AssetKey, AssetsDefinition). It can also acceptJobDefinition(a function decorated with @job is an instance of JobDefinition) and UnresolvedAssetJobDefinition (the return value ofdefine_asset_job()) objects. This parameter will replace job and job_name.
- 
metadata (Optional[Mapping[str, Any]]) – A set of metadata entries that annotate the schedule. Values will be normalized to typed MetadataValue objects. Not currently shown in the UI but available at runtime via ScheduleEvaluationContext.repository_def.get_schedule_def(<name>).metadata. 
 - propertycron_schedule [source]
- The cron schedule representing when this schedule will be evaluated. - Type: Union[str, Sequence[str]] 
 - propertydefault_status [source]
- The default status for this schedule when it is first loaded in a code location. - Type: DefaultScheduleStatus 
 - propertydescription [source]
- A description for this schedule. - Type: Optional[str] 
 - propertyenvironment_vars [source]
- deprecatedThis API will be removed in version 2.0. Setting this property no longer has any effect.. Environment variables to export to the cron schedule. Type: Mapping[str, str] 
 - propertyexecution_timezone [source]
- The timezone in which this schedule will be evaluated. - Type: Optional[str] 
 - propertyjob [source]
- The job that is targeted by this schedule. - Type: Union[JobDefinition, UnresolvedAssetJobDefinition] 
 - propertyjob_name [source]
- The name of the job targeted by this schedule. - Type: str 
 - propertymetadata [source]
- The metadata for this schedule. - Type: Mapping[str, str] 
 - propertyname [source]
- The name of the schedule. - Type: str 
 - propertyrequired_resource_keys [source]
- The set of keys for resources that must be provided to this schedule. - Type: Set[str] 
 - propertytags [source]
- The tags for this schedule. - Type: Mapping[str, str] 
 
- 
- classdagster.ScheduleEvaluationContext [source]
- The context object available as the first argument to various functions defined on a - dagster.ScheduleDefinition.- A - ScheduleEvaluationContextobject is passed as the first argument to- run_config_fn,- tags_fn, and- should_execute.- Users should not instantiate this object directly. To construct a - ScheduleEvaluationContextfor testing purposes, use- dagster.build_schedule_context().- Example: - from dagster import schedule, ScheduleEvaluationContext
 @schedule
 def the_schedule(context: ScheduleEvaluationContext):
 ...- propertyinstance [source]
- The current - DagsterInstance.- Type: DagsterInstance 
 - propertyresources [source]
- Mapping of resource key to resource definition to be made available during schedule execution. 
 - propertyscheduled_execution_time [source]
- The time in which the execution was scheduled to happen. May differ slightly from both the actual execution time and the time at which the run config is computed. 
 
- dagster.build_schedule_context [source]
- Builds schedule execution context using the provided parameters. - The instance provided to - build_schedule_contextmust be persistent;- DagsterInstance.ephemeral()will result in an error.- Parameters: - instance (Optional[DagsterInstance]) – The Dagster instance configured to run the schedule.
- scheduled_execution_time (datetime) – The time in which the execution was scheduled to happen. May differ slightly from both the actual execution time and the time at which the run config is computed.
 - Examples: - context = build_schedule_context(instance)
- dagster.build_schedule_from_partitioned_job [source]
- Creates a schedule from a job that targets time window-partitioned or statically-partitioned assets. The job can also be multi-partitioned, as long as one of the partition dimensions is time-partitioned. - The schedule executes at the cadence specified by the time partitioning of the job or assets. Example:- ######################################
 # Job that targets partitioned assets
 ######################################
 from dagster import (
 DailyPartitionsDefinition,
 asset,
 build_schedule_from_partitioned_job,
 define_asset_job,
 Definitions,
 )
 @asset(partitions_def=DailyPartitionsDefinition(start_date="2020-01-01"))
 def asset1():
 ...
 asset1_job = define_asset_job("asset1_job", selection=[asset1])
 # The created schedule will fire daily
 asset1_job_schedule = build_schedule_from_partitioned_job(asset1_job)
 Definitions(assets=[asset1], schedules=[asset1_job_schedule])
 ################
 # Non-asset job
 ################
 from dagster import DailyPartitionsDefinition, build_schedule_from_partitioned_job, jog
 @job(partitions_def=DailyPartitionsDefinition(start_date="2020-01-01"))
 def do_stuff_partitioned():
 ...
 # The created schedule will fire daily
 do_stuff_partitioned_schedule = build_schedule_from_partitioned_job(
 do_stuff_partitioned,
 )
 Definitions(schedules=[do_stuff_partitioned_schedule])
- dagster._core.scheduler.DagsterDaemonScheduler Scheduler [source]
- Default scheduler implementation that submits runs from the long-lived - dagster-daemonprocess. Periodically checks each running schedule for execution times that don’t yet have runs and launches them.
Sensors
Sensors are typically used to poll, listen, and respond to external events. For example, you could configure a sensor to run a job or materialize an asset in response to specific events.
- @dagster.sensor [source]
- Creates a sensor where the decorated function is used as the sensor’s evaluation function. - The decorated function may: - Return a RunRequest object.
- Return a list of RunRequest objects.
- Return a SkipReason object, providing a descriptive message of why no runs were requested.
- Return nothing (skipping without providing a reason)
- Yield a SkipReason or yield one or more RunRequest objects.
Takes a SensorEvaluationContext.
 - Parameters: - name (Optional[str]) – The name of the sensor. Defaults to the name of the decorated function.
- minimum_interval_seconds (Optional[int]) – The minimum number of seconds that will elapse between sensor evaluations.
- description (Optional[str]) – A human-readable description of the sensor.
- job (Optional[Union[GraphDefinition, JobDefinition, UnresolvedAssetJobDefinition]]) – The job to be executed when the sensor fires.
- jobs (Optional[Sequence[Union[GraphDefinition, JobDefinition, UnresolvedAssetJobDefinition]]]) – A list of jobs to be executed when the sensor fires.
- default_status (DefaultSensorStatus) – Whether the sensor starts as running or not. The default status can be overridden from the Dagster UI or via the GraphQL API.
- asset_selection (Optional[Union[str, Sequence[str], Sequence[AssetKey], Sequence[Union[AssetsDefinition, SourceAsset]], AssetSelection]]) – An asset selection to launch a run for if the sensor condition is met. This can be provided instead of specifying a job.
- required_resource_keys (Optional[set[str]]) – A set of resource keys that must be available on the context when the sensor evaluation function runs. Use this to specify resources your sensor function depends on.
- tags (Optional[Mapping[str, str]]) – A set of key-value tags that annotate the sensor and can be used for searching and filtering in the UI.
- metadata (Optional[Mapping[str, object]]) – A set of metadata entries that annotate the sensor. Values will be normalized to typed MetadataValue objects.
- target (Optional[Union[CoercibleToAssetSelection, AssetsDefinition, JobDefinition, UnresolvedAssetJobDefinition]]) – The target that the sensor will execute. It can take AssetSelectionobjects and anything coercible to it (e.g. str, Sequence[str], AssetKey, AssetsDefinition). It can also acceptJobDefinition(a function decorated with @job is an instance of JobDefinition) and UnresolvedAssetJobDefinition (the return value ofdefine_asset_job()) objects. This is a parameter that will replace job, jobs, and asset_selection.
- owners (Optional[Sequence[str]]) – beta A sequence of strings identifying the owners of the sensor.
 
- classdagster.SensorDefinition [source]
- Define a sensor that initiates a set of runs based on some external state. - Parameters: - 
evaluation_fn (Callable[[SensorEvaluationContext]]) – The core evaluation function for the sensor, which is run at an interval to determine whether a run should be launched or not. Takes a SensorEvaluationContext.
- 
name (Optional[str]) – The name of the sensor to create. Defaults to name of evaluation_fn 
- 
minimum_interval_seconds (Optional[int]) – The minimum number of seconds that will elapse between sensor evaluations. 
- 
description (Optional[str]) – A human-readable description of the sensor. 
- 
job (Optional[GraphDefinition, JobDefinition, UnresolvedAssetJob]) – The job to execute when this sensor fires. 
- 
jobs (Optional[Sequence[GraphDefinition, JobDefinition, UnresolvedAssetJob]]) – A list of jobs to execute when this sensor fires. 
- 
default_status (DefaultSensorStatus) – Whether the sensor starts as running or not. The default status can be overridden from the Dagster UI or via the GraphQL API. 
- 
asset_selection (Optional[Union[str, Sequence[str], Sequence[AssetKey], Sequence[Union[AssetsDefinition, SourceAsset]], AssetSelection]]) – An asset selection to launch a run for if the sensor condition is met. This can be provided instead of specifying a job. 
- 
tags (Optional[Mapping[str, str]]) – A set of key-value tags that annotate the sensor and can be used for searching and filtering in the UI. 
- 
metadata (Optional[Mapping[str, object]]) – A set of metadata entries that annotate the sensor. Values will be normalized to typed MetadataValue objects. Not currently shown in the UI but available at runtime via SensorEvaluationContext.repository_def.get_sensor_def(<name>).metadata. 
- 
target (Optional[Union[CoercibleToAssetSelection, AssetsDefinition, JobDefinition, UnresolvedAssetJobDefinition]]) – The target that the sensor will execute. It can take AssetSelectionobjects and anything coercible to it (e.g. str, Sequence[str], AssetKey, AssetsDefinition). It can also acceptJobDefinition(a function decorated with @job is an instance of JobDefinition) and UnresolvedAssetJobDefinition (the return value ofdefine_asset_job()) objects. This is a parameter that will replace job, jobs, and asset_selection.
 - propertydefault_status [source]
- The default status for this sensor when it is first loaded in a code location. - Type: DefaultSensorStatus 
 - propertydescription [source]
- A description for this sensor. - Type: Optional[str] 
 - propertyjob [source]
- The job that is targeted by this schedule. - Type: Union[GraphDefinition, JobDefinition, UnresolvedAssetJobDefinition] 
 - propertyjob_name [source]
- The name of the job that is targeted by this sensor. - Type: Optional[str] 
 - propertyjobs [source]
- A list of jobs that are targeted by this schedule. - Type: List[Union[GraphDefinition, JobDefinition, UnresolvedAssetJobDefinition]] 
 - propertyminimum_interval_seconds [source]
- The minimum number of seconds between sequential evaluations of this sensor. - Type: Optional[int] 
 - propertyname [source]
- The name of this sensor. - Type: str 
 - propertyrequired_resource_keys [source]
- The set of keys for resources that must be provided to this sensor. - Type: Set[str] 
 
- 
- classdagster.SensorEvaluationContext [source]
- The context object available as the argument to the evaluation function of a - dagster.SensorDefinition.- Users should not instantiate this object directly. To construct a SensorEvaluationContext for testing purposes, use - dagster. build_sensor_context().- Parameters: - instance_ref (Optional[InstanceRef]) – The serialized instance configured to run the schedule
- cursor (Optional[str]) – The cursor, passed back from the last sensor evaluation via the cursor attribute of SkipReason and RunRequest
- last_tick_completion_time (float) – The last time that the sensor was evaluated (UTC).
- last_run_key (str) – DEPRECATED The run key of the RunRequest most recently created by this sensor. Use the preferred cursor attribute instead.
- log_key (Optional[List[str]]) – The log key to use for this sensor tick.
- repository_name (Optional[str]) – The name of the repository that the sensor belongs to.
- repository_def (Optional[RepositoryDefinition]) – The repository or that the sensor belongs to. If needed by the sensor top-level resource definitions will be pulled from this repository. You can provide either this or definitions.
- instance (Optional[DagsterInstance]) – The deserialized instance can also be passed in directly (primarily useful in testing contexts).
- definitions (Optional[Definitions]) – Definitions object that the sensor is defined in. If needed by the sensor, top-level resource definitions will be pulled from these definitions. You can provide either this or repository_def.
- resources (Optional[Dict[str, Any]]) – A dict of resource keys to resource definitions to be made available during sensor execution.
- last_sensor_start_time (float) – The last time that the sensor was started (UTC).
- code_location_origin (Optional[CodeLocationOrigin]) – The code location that the sensor is in.
 - Example: - from dagster import sensor, SensorEvaluationContext
 @sensor
 def the_sensor(context: SensorEvaluationContext):
 ...- update_cursor [source]
- Updates the cursor value for this sensor, which will be provided on the context for the next sensor evaluation. - This can be used to keep track of progress and avoid duplicate work across sensor evaluations. - Parameters: cursor (Optional[str]) 
 - propertycursor [source]
- The cursor value for this sensor, which was set in an earlier sensor evaluation. 
 - propertyinstance [source]
- The current DagsterInstance. - Type: DagsterInstance 
 - propertyis_first_tick_since_sensor_start [source]
- Flag representing if this is the first tick since the sensor was started. 
 - propertylast_run_key [source]
- The run key supplied to the most recent RunRequest produced by this sensor. - Type: Optional[str] 
 - propertylast_sensor_start_time [source]
- Timestamp representing the last time this sensor was started. Can be used in concert with last_tick_completion_time to determine if this is the first tick since the sensor was started. - Type: Optional[float] 
 - propertylast_tick_completion_time [source]
- Timestamp representing the last time this sensor completed an evaluation. - Type: Optional[float] 
 - propertyrepository_def [source]
- The RepositoryDefinition that this sensor resides in. - Type: Optional[RepositoryDefinition] 
 - propertyrepository_name [source]
- The name of the repository that this sensor resides in. - Type: Optional[str] 
 - propertyresources [source]
- A mapping from resource key to instantiated resources for this sensor. - Type: Resources 
 
- dagster.build_sensor_context [source]
- Builds sensor execution context using the provided parameters. - This function can be used to provide a context to the invocation of a sensor definition.If provided, the dagster instance must be persistent; DagsterInstance.ephemeral() will result in an error. - Parameters: - instance (Optional[DagsterInstance]) – The dagster instance configured to run the sensor.
- cursor (Optional[str]) – A cursor value to provide to the evaluation of the sensor.
- repository_name (Optional[str]) – The name of the repository that the sensor belongs to.
- repository_def (Optional[RepositoryDefinition]) – The repository that the sensor belongs to. If needed by the sensor top-level resource definitions will be pulled from this repository. You can provide either this or definitions.
- resources (Optional[Mapping[str, ResourceDefinition]]) – A set of resource definitions to provide to the sensor. If passed, these will override any resource definitions provided by the repository.
- definitions (Optional[Definitions]) – Definitions object that the sensor is defined in. If needed by the sensor, top-level resource definitions will be pulled from these definitions. You can provide either this or repository_def.
- last_sensor_start_time (Optional[float]) – The last time the sensor was started.
 - Examples: - context = build_sensor_context()
 my_sensor(context)
- @dagster.asset_sensor [source]
- Creates an asset sensor where the decorated function is used as the asset sensor’s evaluation function. - If the asset has been materialized multiple times between since the last sensor tick, the evaluation function will only be invoked once, with the latest materialization. - The decorated function may: - Return a RunRequest object.
- Return a list of RunRequest objects.
- Return a SkipReason object, providing a descriptive message of why no runs were requested.
- Return nothing (skipping without providing a reason)
- Yield a SkipReason or yield one or more RunRequest objects.
Takes a SensorEvaluationContextand an EventLogEntry corresponding to an AssetMaterialization event.
 - Parameters: - asset_key (AssetKey) – The asset_key this sensor monitors.
- name (Optional[str]) – The name of the sensor. Defaults to the name of the decorated function.
- minimum_interval_seconds (Optional[int]) – The minimum number of seconds that will elapse between sensor evaluations.
- description (Optional[str]) – A human-readable description of the sensor.
- job (Optional[Union[GraphDefinition, JobDefinition, UnresolvedAssetJobDefinition]]) – The job to be executed when the sensor fires.
- jobs (Optional[Sequence[Union[GraphDefinition, JobDefinition, UnresolvedAssetJobDefinition]]]) – A list of jobs to be executed when the sensor fires.
- default_status (DefaultSensorStatus) – Whether the sensor starts as running or not. The default status can be overridden from the Dagster UI or via the GraphQL API.
- tags (Optional[Mapping[str, str]]) – A set of key-value tags that annotate the sensor and can be used for searching and filtering in the UI. Values that are not already strings will be serialized as JSON.
- metadata (Optional[Mapping[str, object]]) – A set of metadata entries that annotate the sensor. Values will be normalized to typed MetadataValue objects.
 - Example: - from dagster import AssetKey, EventLogEntry, SensorEvaluationContext, asset_sensor
 @asset_sensor(asset_key=AssetKey("my_table"), job=my_job)
 def my_asset_sensor(context: SensorEvaluationContext, asset_event: EventLogEntry):
 return RunRequest(
 run_key=context.cursor,
 run_config={
 "ops": {
 "read_materialization": {
 "config": {
 "asset_key": asset_event.dagster_event.asset_key.path,
 }
 }
 }
 },
 )
- @dagster.multi_asset_sensor [source]
- Creates an asset sensor that can monitor multiple assets. - The decorated function is used as the asset sensor’s evaluation function. The decorated function may: - Return a RunRequest object.
- Return a list of RunRequest objects.
- Return a SkipReason object, providing a descriptive message of why no runs were requested.
- Return nothing (skipping without providing a reason)
- Yield a SkipReason or yield one or more RunRequest objects.
Takes a MultiAssetSensorEvaluationContext.
 - Parameters: - monitored_assets (Union[Sequence[AssetKey], AssetSelection]) – The assets this sensor monitors. If an AssetSelection object is provided, it will only apply to assets within the Definitions that this sensor is part of.
- name (Optional[str]) – The name of the sensor. Defaults to the name of the decorated function.
- minimum_interval_seconds (Optional[int]) – The minimum number of seconds that will elapse between sensor evaluations.
- description (Optional[str]) – A human-readable description of the sensor.
- job (Optional[Union[GraphDefinition, JobDefinition, UnresolvedAssetJobDefinition]]) – The job to be executed when the sensor fires.
- jobs (Optional[Sequence[Union[GraphDefinition, JobDefinition, UnresolvedAssetJobDefinition]]]) – A list of jobs to be executed when the sensor fires.
- default_status (DefaultSensorStatus) – Whether the sensor starts as running or not. The default status can be overridden from the Dagster UI or via the GraphQL API.
- request_assets (Optional[AssetSelection]) – An asset selection to launch a run for if the sensor condition is met. This can be provided instead of specifying a job.
- tags (Optional[Mapping[str, str]]) – A set of key-value tags that annotate the sensor and can be used for searching and filtering in the UI.
- metadata (Optional[Mapping[str, object]]) – A set of metadata entries that annotate the sensor. Values will be normalized to typed MetadataValue objects.
 
- @dagster.run_status_sensor [source]
- Creates a sensor that reacts to a given status of job execution, where the decorated function will be run when a job is at the given status. - Takes a - RunStatusSensorContext.- Parameters: - run_status (DagsterRunStatus) – The status of run execution which will be monitored by the sensor.
- name (Optional[str]) – The name of the sensor. Defaults to the name of the decorated function.
- minimum_interval_seconds (Optional[int]) – The minimum number of seconds that will elapse between sensor evaluations.
- description (Optional[str]) – A human-readable description of the sensor.
- monitored_jobs (Optional[List[Union[JobDefinition, GraphDefinition, UnresolvedAssetJobDefinition, RepositorySelector, JobSelector, CodeLocationSelector]]]) – Jobs in the current code locations that will be monitored by this sensor. Defaults to None, which means the alert will be sent when any job in the code location matches the requested run_status. Jobs in external repositories can be monitored by using RepositorySelector or JobSelector.
- monitor_all_code_locations (Optional[bool]) – If set to True, the sensor will monitor all runs in the Dagster deployment. If set to True, an error will be raised if you also specify monitored_jobs or job_selection. Defaults to False.
- job_selection (Optional[List[Union[JobDefinition, GraphDefinition, RepositorySelector, JobSelector, CodeLocationSelector]]]) – deprecated (deprecated in favor of monitored_jobs) Jobs in the current code location that will be monitored by this sensor. Defaults to None, which means the alert will be sent when any job in the code location matches the requested run_status.
- default_status (DefaultSensorStatus) – Whether the sensor starts as running or not. The default status can be overridden from the Dagster UI or via the GraphQL API.
- request_job (Optional[Union[GraphDefinition, JobDefinition, UnresolvedAssetJobDefinition]]) – The job that should be executed if a RunRequest is yielded from the sensor.
- request_jobs (Optional[Sequence[Union[GraphDefinition, JobDefinition, UnresolvedAssetJobDefinition]]]) – A list of jobs to be executed if RunRequests are yielded from the sensor.
- monitor_all_repositories (Optional[bool]) – deprecated (deprecated in favor of monitor_all_code_locations) If set to True, the sensor will monitor all runs in the Dagster instance. If set to True, an error will be raised if you also specify monitored_jobs or job_selection. Defaults to False.
- tags (Optional[Mapping[str, str]]) – A set of key-value tags that annotate the sensor and can be used for searching and filtering in the UI.
- metadata (Optional[Mapping[str, object]]) – A set of metadata entries that annotate the sensor. Values will be normalized to typed MetadataValue objects.
 
- @dagster.run_failure_sensor [source]
- Creates a sensor that reacts to job failure events, where the decorated function will be run when a run fails. - Takes a - RunFailureSensorContext.- Parameters: - name (Optional[str]) – The name of the job failure sensor. Defaults to the name of the decorated function.
- minimum_interval_seconds (Optional[int]) – The minimum number of seconds that will elapse between sensor evaluations.
- description (Optional[str]) – A human-readable description of the sensor.
- monitored_jobs (Optional[List[Union[JobDefinition, GraphDefinition, UnresolvedAssetJobDefinition, RepositorySelector, JobSelector, CodeLocationSelector]]]) – The jobs in the current repository that will be monitored by this failure sensor. Defaults to None, which means the alert will be sent when any job in the current repository fails.
- monitor_all_code_locations (bool) – If set to True, the sensor will monitor all runs in the Dagster deployment. If set to True, an error will be raised if you also specify monitored_jobs or job_selection. Defaults to False.
- job_selection (Optional[List[Union[JobDefinition, GraphDefinition, RepositorySelector, JobSelector, CodeLocationSelector]]]) – deprecated (deprecated in favor of monitored_jobs) The jobs in the current repository that will be monitored by this failure sensor. Defaults to None, which means the alert will be sent when any job in the repository fails.
- default_status (DefaultSensorStatus) – Whether the sensor starts as running or not. The default status can be overridden from the Dagster UI or via the GraphQL API.
- request_job (Optional[Union[GraphDefinition, JobDefinition, UnresolvedAssetJob]]) – The job a RunRequest should execute if yielded from the sensor.
- request_jobs (Optional[Sequence[Union[GraphDefinition, JobDefinition, UnresolvedAssetJob]]]) – A list of jobs to be executed if RunRequests are yielded from the sensor.
- monitor_all_repositories (bool) – deprecated (deprecated in favor of monitor_all_code_locations) If set to True, the sensor will monitor all runs in the Dagster instance. If set to True, an error will be raised if you also specify monitored_jobs or job_selection. Defaults to False.
- tags (Optional[Mapping[str, str]]) – A set of key-value tags that annotate the sensor and can be used for searching and filtering in the UI.
- metadata (Optional[Mapping[str, object]]) – A set of metadata entries that annotate the sensor. Values will be normalized to typed MetadataValue objects.
 
- classdagster.AssetSensorDefinition [source]
- Define an asset sensor that initiates a set of runs based on the materialization of a given asset. - If the asset has been materialized multiple times between since the last sensor tick, the evaluation function will only be invoked once, with the latest materialization. - Parameters: - 
name (str) – The name of the sensor to create. 
- 
asset_key (AssetKey) – The asset_key this sensor monitors. 
- 
asset_materialization_fn (Callable[[SensorEvaluationContext, EventLogEntry], Union[Iterator[Union[RunRequest, SkipReason]], RunRequest, SkipReason]]) – The core evaluation function for the sensor, which is run at an interval to determine whether a run should be launched or not. Takes a SensorEvaluationContextand an EventLogEntry corresponding to an AssetMaterialization event.
- 
minimum_interval_seconds (Optional[int]) – The minimum number of seconds that will elapse between sensor evaluations. 
- 
description (Optional[str]) – A human-readable description of the sensor. 
- 
job (Optional[Union[GraphDefinition, JobDefinition, UnresolvedAssetJobDefinition]]) – The job object to target with this sensor. 
- 
jobs (Optional[Sequence[Union[GraphDefinition, JobDefinition, UnresolvedAssetJobDefinition]]]) – A list of jobs to be executed when the sensor fires. 
- 
tags (Optional[Mapping[str, str]]) – A set of key-value tags that annotate the sensor and can be used for searching and filtering in the UI. 
- 
metadata (Optional[Mapping[str, object]]) – A set of metadata entries that annotate the sensor. Values will be normalized to typed MetadataValue objects. 
- 
default_status (DefaultSensorStatus) – Whether the sensor starts as running or not. The default status can be overridden from the Dagster UI or via the GraphQL API. 
 - propertyasset_key [source]
- The key of the asset targeted by this sensor. - Type: AssetKey 
 
- 
- classdagster.MultiAssetSensorDefinition [source]
- supersededThis API has been superseded. For most use cases, Declarative Automation should be used instead of multi_asset_sensors to monitor the status of upstream assets and launch runs in response. In cases where side effects are required, or a specific job must be targeted for execution, multi_asset_sensors may be used.. Define an asset sensor that initiates a set of runs based on the materialization of a list of assets. Users should not instantiate this object directly. To construct a MultiAssetSensorDefinition, use dagster. multi_asset_sensor().Parameters: - 
name (str) – The name of the sensor to create. 
- 
asset_keys (Sequence[AssetKey]) – The asset_keys this sensor monitors. 
- 
asset_materialization_fn (Callable[[MultiAssetSensorEvaluationContext], Union[Iterator[Union[RunRequest, SkipReason]], RunRequest, SkipReason]]) – The core evaluation function for the sensor, which is run at an interval to determine whether a run should be launched or not. Takes a MultiAssetSensorEvaluationContext.
- 
minimum_interval_seconds (Optional[int]) – The minimum number of seconds that will elapse between sensor evaluations. 
- 
description (Optional[str]) – A human-readable description of the sensor. 
- 
job (Optional[Union[GraphDefinition, JobDefinition, UnresolvedAssetJobDefinition]]) – The job object to target with this sensor. 
- 
jobs (Optional[Sequence[Union[GraphDefinition, JobDefinition, UnresolvedAssetJobDefinition]]]) – A list of jobs to be executed when the sensor fires. 
- 
default_status (DefaultSensorStatus) – Whether the sensor starts as running or not. The default status can be overridden from the Dagster UI or via the GraphQL API. 
- 
request_assets (Optional[AssetSelection]) – an asset selection to launch a run for if the sensor condition is met. This can be provided instead of specifying a job. 
- 
tags (Optional[Mapping[str, str]]) – A set of key-value tags that annotate the sensor and can be used for searching and filtering in the UI. 
- 
metadata (Optional[Mapping[str, object]]) – A set of metadata entries that annotate the sensor. Values will be normalized to typed MetadataValue objects. 
 
- 
- classdagster.RunStatusSensorDefinition [source]
- Define a sensor that reacts to a given status of job execution, where the decorated function will be evaluated when a run is at the given status. - Parameters: - name (str) – The name of the sensor. Defaults to the name of the decorated function.
- run_status (DagsterRunStatus) – The status of a run which will be monitored by the sensor.
- run_status_sensor_fn (Callable[[RunStatusSensorContext], Union[SkipReason, DagsterRunReaction]]) – The core evaluation function for the sensor. Takes a RunStatusSensorContext.
- minimum_interval_seconds (Optional[int]) – The minimum number of seconds that will elapse between sensor evaluations.
- description (Optional[str]) – A human-readable description of the sensor.
- monitored_jobs (Optional[List[Union[JobDefinition, GraphDefinition, UnresolvedAssetJobDefinition, JobSelector, RepositorySelector, CodeLocationSelector]]]) – The jobs in the current repository that will be monitored by this sensor. Defaults to None, which means the alert will be sent when any job in the repository fails.
- monitor_all_code_locations (bool) – If set to True, the sensor will monitor all runs in the Dagster deployment. If set to True, an error will be raised if you also specify monitored_jobs or job_selection. Defaults to False.
- default_status (DefaultSensorStatus) – Whether the sensor starts as running or not. The default status can be overridden from the Dagster UI or via the GraphQL API.
- request_job (Optional[Union[GraphDefinition, JobDefinition]]) – The job a RunRequest should execute if yielded from the sensor.
- tags (Optional[Mapping[str, str]]) – A set of key-value tags that annotate the sensor and can be used for searching and filtering in the UI.
- metadata (Optional[Mapping[str, object]]) – A set of metadata entries that annotate the sensor. Values will be normalized to typed MetadataValue objects.
- request_jobs (Optional[Sequence[Union[GraphDefinition, JobDefinition]]]) – A list of jobs to be executed if RunRequests are yielded from the sensor.
 
- classdagster.RunStatusSensorContext [source]
- The - contextobject available to a decorated function of- run_status_sensor.- propertydagster_event [source]
- The event associated with the job run status. 
 - propertydagster_run [source]
- The run of the job. 
 - propertyinstance [source]
- The current instance. 
 - propertylog [source]
- The logger for the current sensor evaluation. 
 - propertypartition_key [source]
- The partition key of the relevant run. - Type: Optional[str] 
 - propertysensor_name [source]
- The name of the sensor. 
 
- classdagster.RunFailureSensorContext [source]
- The - contextobject available to a decorated function of- run_failure_sensor.- Parameters: - sensor_name (str) – the name of the sensor.
- dagster_run (DagsterRun) – the failed run.
 - get_step_failure_events [source]
- The step failure event for each step in the run that failed. - Examples: - error_strings_by_step_key = {
 # includes the stack trace
 event.step_key: event.event_specific_data.error.to_string()
 for event in context.get_step_failure_events()
 }
 - propertyfailure_event [source]
- The run failure event. - If the run failed because of an error inside a step, get_step_failure_events will have more details on the step failure. 
 
- dagster.build_run_status_sensor_context [source]
- Builds run status sensor context from provided parameters. - This function can be used to provide the context argument when directly invoking a function decorated with @run_status_sensor or @run_failure_sensor, such as when writing unit tests. - Parameters: - sensor_name (str) – The name of the sensor the context is being constructed for.
- dagster_event (DagsterEvent) – A DagsterEvent with the same event type as the one that triggers the run_status_sensor
- dagster_instance (DagsterInstance) – The dagster instance configured for the context.
- dagster_run (DagsterRun) – DagsterRun object from running a job
- resources (Optional[Mapping[str, object]]) – A dictionary of resources to be made available to the sensor.
- repository_def (Optional[RepositoryDefinition]) – beta The repository that the sensor belongs to.
 - Examples: - instance = DagsterInstance.ephemeral()
 result = my_job.execute_in_process(instance=instance)
 dagster_run = result.dagster_run
 dagster_event = result.get_job_success_event() # or get_job_failure_event()
 context = build_run_status_sensor_context(
 sensor_name="run_status_sensor_to_invoke",
 dagster_instance=instance,
 dagster_run=dagster_run,
 dagster_event=dagster_event,
 )
 run_status_sensor_to_invoke(context)
- classdagster.SensorResult [source]
- The result of a sensor evaluation. - Parameters: - run_requests (Optional[Sequence[RunRequest]]) – A list of run requests to be executed.
- skip_reason (Optional[Union[str, SkipReason]]) – A skip message indicating why sensor evaluation was skipped.
- cursor (Optional[str]) – The cursor value for this sensor, which will be provided on the context for the next sensor evaluation.
- dynamic_partitions_requests (Optional[Sequence[Union[DeleteDynamicPartitionsRequest, AddDynamicPartitionsRequest]]]) – A list of dynamic partition requests to request dynamic partition addition and deletion. Run requests will be evaluated using the state of the partitions with these changes applied. We recommend limiting partition additions and deletions to a maximum of 25K partitions per sensor evaluation, as this is the maximum recommended partition limit per asset.
- asset_events (Optional[Sequence[Union[AssetObservation, AssetMaterialization, AssetCheckEvaluation]]]) – A list of materializations, observations, and asset check evaluations that the system will persist on your behalf at the end of sensor evaluation. These events will be not be associated with any particular run, but will be queryable and viewable in the asset catalog.
 
- classdagster.AddDynamicPartitionsRequest [source]
- A request to add partitions to a dynamic partitions definition, to be evaluated by a sensor or schedule. 
- classdagster.DeleteDynamicPartitionsRequest [source]
- A request to delete partitions to a dynamic partitions definition, to be evaluated by a sensor or schedule.