Skip to content

Commit

Permalink
add launched by type to gql schema
Browse files Browse the repository at this point in the history
  • Loading branch information
jamiedemaria committed Oct 14, 2024
1 parent 169d673 commit 680b7b1
Show file tree
Hide file tree
Showing 6 changed files with 126 additions and 1 deletion.

Some generated files are not rendered by default. Learn more about how customized files appear on GitHub.

40 changes: 40 additions & 0 deletions js_modules/dagster-ui/packages/ui-core/src/graphql/types.ts

Some generated files are not rendered by default. Learn more about how customized files appear on GitHub.

30 changes: 30 additions & 0 deletions python_modules/dagster-graphql/dagster_graphql/schema/backfill.py
Original file line number Diff line number Diff line change
Expand Up @@ -23,6 +23,11 @@
from dagster._core.storage.tags import (
ASSET_PARTITION_RANGE_END_TAG,
ASSET_PARTITION_RANGE_START_TAG,
AUTO_MATERIALIZE_TAG,
AUTO_OBSERVE_TAG,
SCHEDULE_NAME_TAG,
SENSOR_NAME_TAG,
USER_TAG,
TagType,
get_tag_type,
)
Expand All @@ -47,6 +52,7 @@
GrapheneUnauthorizedError,
create_execution_params_error_types,
)
from dagster_graphql.schema.launched_by import GrapheneLaunchedBy
from dagster_graphql.schema.pipelines.config import GrapheneRunConfigValidationInvalid
from dagster_graphql.schema.pipelines.status import GrapheneRunStatus
from dagster_graphql.schema.runs_feed import GrapheneRunsFeedEntry
Expand Down Expand Up @@ -385,6 +391,7 @@ class Meta:
assetCheckSelection = graphene.List(
graphene.NonNull("dagster_graphql.schema.asset_checks.GrapheneAssetCheckHandle")
)
launchedBy = graphene.NonNull(GrapheneLaunchedBy)

def __init__(self, backfill_job: PartitionBackfill):
self._backfill_job = check.inst_param(backfill_job, "backfill_job", PartitionBackfill)
Expand Down Expand Up @@ -520,6 +527,29 @@ def resolve_tags(self, _graphene_info: ResolveInfo):
if get_tag_type(key) != TagType.HIDDEN
]

def resolve_launchedBy(self, _graphene_info: ResolveInfo):
"""Determines which value should be shown in the Launched by column in the UI. This value is
deetermined based on the tags of the backfill, not the source of the backfill as stored in the DB. Thus, some
backfills may have different launchedBy values from source columns.
"""
if self.tags.get(USER_TAG):
return GrapheneLaunchedBy(kind="user", value=self.tags[USER_TAG])
if self.tags.get(SCHEDULE_NAME_TAG):
return GrapheneLaunchedBy(kind="schedule", value=self.tags[SCHEDULE_NAME_TAG])
if self.tags.get(SENSOR_NAME_TAG):
return GrapheneLaunchedBy(kind="sensor", value=self.tags[SENSOR_NAME_TAG])
if self.tage.get(AUTO_MATERIALIZE_TAG):
return GrapheneLaunchedBy(
kind="auto-materialize", value=self.tags[AUTO_MATERIALIZE_TAG]
)
if self.tags.get("dagster/created_by") == "auto_materialize":
return GrapheneLaunchedBy(
kind="auto-materialize", value=self.tags["dagster/created_by"]
)
if self.tags.get(AUTO_OBSERVE_TAG):
return GrapheneLaunchedBy(kind="auto-observe", value=self.tags[AUTO_OBSERVE_TAG])
return GrapheneLaunchedBy(kind="manual", value="")

def resolve_runStatus(self, _graphene_info: ResolveInfo) -> GrapheneRunStatus:
return GrapheneBulkActionStatus(self.status).to_dagster_run_status()

Expand Down
Original file line number Diff line number Diff line change
@@ -0,0 +1,10 @@
import graphene


class GrapheneLaunchedBy(graphene.ObjectType):
kind = graphene.NonNull(graphene.String)
value = graphene.NonNull(graphene.String)

def __init__(self, kind: str, value: str):
self.kind = kind
self.value = value
Original file line number Diff line number Diff line change
Expand Up @@ -19,7 +19,17 @@
RunRecord,
RunsFilter,
)
from dagster._core.storage.tags import REPOSITORY_LABEL_TAG, RUN_METRIC_TAGS, TagType, get_tag_type
from dagster._core.storage.tags import (
AUTO_MATERIALIZE_TAG,
AUTO_OBSERVE_TAG,
REPOSITORY_LABEL_TAG,
RUN_METRIC_TAGS,
SCHEDULE_NAME_TAG,
SENSOR_NAME_TAG,
USER_TAG,
TagType,
get_tag_type,
)
from dagster._core.workspace.permissions import Permissions
from dagster._utils.tags import get_boolean_tag_value
from dagster._utils.yaml_utils import dump_run_config_yaml
Expand Down Expand Up @@ -51,6 +61,7 @@
)
from dagster_graphql.schema.execution import GrapheneExecutionPlan
from dagster_graphql.schema.inputs import GrapheneAssetKeyInput
from dagster_graphql.schema.launched_by import GrapheneLaunchedBy
from dagster_graphql.schema.logs.compute_logs import GrapheneCapturedLogs, from_captured_log_data
from dagster_graphql.schema.logs.events import (
GrapheneDagsterRunEvent,
Expand Down Expand Up @@ -391,6 +402,7 @@ class GrapheneRun(graphene.ObjectType):
rootConcurrencyKeys = graphene.List(graphene.NonNull(graphene.String))
hasUnconstrainedRootNodes = graphene.NonNull(graphene.Boolean)
hasRunMetricsEnabled = graphene.NonNull(graphene.Boolean)
launchedBy = graphene.NonNull(GrapheneLaunchedBy)

class Meta:
interfaces = (GraphenePipelineRun, GrapheneRunsFeedEntry)
Expand Down Expand Up @@ -529,6 +541,29 @@ def resolve_tags(self, _graphene_info: ResolveInfo):
if get_tag_type(key) != TagType.HIDDEN
]

def resolve_launchedBy(self, _graphene_info: ResolveInfo):
"""Determines which value should be shown in the Launched by column in the UI. This value is
deetermined based on the tags of the run, not the source of the run as stored in the DB. Thus, some
runs may have different launchedBy values from source columns.
"""
if self.tags.get(USER_TAG):
return GrapheneLaunchedBy(kind="user", value=self.tags[USER_TAG])
if self.tags.get(SCHEDULE_NAME_TAG):
return GrapheneLaunchedBy(kind="schedule", value=self.tags[SCHEDULE_NAME_TAG])
if self.tags.get(SENSOR_NAME_TAG):
return GrapheneLaunchedBy(kind="sensor", value=self.tags[SENSOR_NAME_TAG])
if self.tage.get(AUTO_MATERIALIZE_TAG):
return GrapheneLaunchedBy(
kind="auto-materialize", value=self.tags[AUTO_MATERIALIZE_TAG]
)
if self.tags.get("dagster/created_by") == "auto_materialize":
return GrapheneLaunchedBy(
kind="auto-materialize", value=self.tags["dagster/created_by"]
)
if self.tags.get(AUTO_OBSERVE_TAG):
return GrapheneLaunchedBy(kind="auto-observe", value=self.tags[AUTO_OBSERVE_TAG])
return GrapheneLaunchedBy(kind="manual", value="")

def resolve_rootRunId(self, _graphene_info: ResolveInfo):
return self.dagster_run.root_run_id

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -2,6 +2,7 @@

from dagster_graphql.schema.asset_key import GrapheneAssetKey
from dagster_graphql.schema.errors import GraphenePythonError
from dagster_graphql.schema.launched_by import GrapheneLaunchedBy
from dagster_graphql.schema.util import non_null_list


Expand All @@ -17,6 +18,7 @@ class GrapheneRunsFeedEntry(graphene.Interface):
assetCheckSelection = graphene.List(
graphene.NonNull("dagster_graphql.schema.asset_checks.GrapheneAssetCheckHandle")
)
launchedBy = graphene.NonNull(GrapheneLaunchedBy)

class Meta:
name = "RunsFeedEntry"
Expand Down

0 comments on commit 680b7b1

Please sign in to comment.