Skip to content

Commit c06ac7f

Browse files
committed
try test again
1 parent a86ae0f commit c06ac7f

File tree

3 files changed

+31
-100
lines changed

3 files changed

+31
-100
lines changed

python_modules/dagster-graphql/dagster_graphql_tests/graphql/test_partition_backfill.py

Lines changed: 0 additions & 61 deletions
Original file line numberDiff line numberDiff line change
@@ -37,7 +37,6 @@
3737
from dagster_graphql.client.query import (
3838
LAUNCH_PARTITION_BACKFILL_MUTATION,
3939
LAUNCH_PIPELINE_EXECUTION_MUTATION,
40-
LAUNCH_PIPELINE_REEXECUTION_MUTATION,
4140
)
4241
from dagster_graphql.test.utils import (
4342
execute_dagster_graphql,
@@ -2303,63 +2302,3 @@ def test_retry_successful_job_backfill(self, graphql_context):
23032302

23042303
assert retried_backfill.tags.get(PARENT_BACKFILL_ID_TAG) == backfill_id
23052304
assert retried_backfill.tags.get(ROOT_BACKFILL_ID_TAG) == backfill_id
2306-
2307-
def test_run_retry_not_part_of_completed_backfill(self, graphql_context):
2308-
# TestLaunchDaemonBackfillFromFailure::test_run_retry_not_part_of_completed_backfill
2309-
repository_selector = infer_repository_selector(graphql_context)
2310-
result = execute_dagster_graphql(
2311-
graphql_context,
2312-
LAUNCH_PARTITION_BACKFILL_MUTATION,
2313-
variables={
2314-
"backfillParams": {
2315-
"selector": {
2316-
"repositorySelector": repository_selector,
2317-
"partitionSetName": "integers_partition_set",
2318-
},
2319-
"partitionNames": ["2", "3", "4", "5"],
2320-
}
2321-
},
2322-
)
2323-
2324-
assert not result.errors
2325-
assert result.data
2326-
assert result.data["launchPartitionBackfill"]["__typename"] == "LaunchBackfillSuccess"
2327-
backfill_id = result.data["launchPartitionBackfill"]["backfillId"]
2328-
2329-
_seed_runs(
2330-
graphql_context,
2331-
[
2332-
(DagsterRunStatus.SUCCESS, "5"),
2333-
(DagsterRunStatus.SUCCESS, "2"),
2334-
(DagsterRunStatus.SUCCESS, "3"),
2335-
(DagsterRunStatus.SUCCESS, "4"),
2336-
(DagsterRunStatus.SUCCESS, "5"),
2337-
(DagsterRunStatus.SUCCESS, "2"),
2338-
(DagsterRunStatus.FAILURE, "3"),
2339-
(DagsterRunStatus.SUCCESS, "4"),
2340-
],
2341-
backfill_id,
2342-
)
2343-
2344-
backfill = graphql_context.instance.get_backfill(backfill_id)
2345-
graphql_context.instance.update_backfill(
2346-
backfill.with_status(BulkActionStatus.COMPLETED_SUCCESS)
2347-
)
2348-
2349-
failed_run = graphql_context.instance.get_runs(
2350-
filters=RunsFilter(statuses=[DagsterRunStatus.FAILURE])
2351-
)[0]
2352-
2353-
retry_run_result = execute_dagster_graphql(
2354-
graphql_context,
2355-
LAUNCH_PIPELINE_REEXECUTION_MUTATION,
2356-
variables={
2357-
"reexecutionParams": {"parentRunId": failed_run.run_id, "strategy": "ALL_STEPS"}
2358-
},
2359-
)
2360-
assert not retry_run_result.errors
2361-
assert retry_run_result.data
2362-
assert (
2363-
retry_run_result.data["launchPipelineReexecution"]["__typename"]
2364-
== "LaunchPipelineReexecutionSuccess"
2365-
)

python_modules/dagster/dagster_tests/daemon_tests/conftest.py

Lines changed: 12 additions & 5 deletions
Original file line numberDiff line numberDiff line change
@@ -64,16 +64,23 @@ def workspace_fixture(instance_module_scoped) -> Iterator[WorkspaceProcessContex
6464
yield workspace_context
6565

6666

67-
@pytest.fixture(name="remote_repo", scope="module")
68-
def remote_repo_fixture(
67+
@pytest.fixture(name="code_location", scope="module")
68+
def code_location_fixture(
6969
workspace_context: WorkspaceProcessContext,
70-
) -> Iterator[RemoteRepository]:
71-
yield cast(
70+
) -> CodeLocation:
71+
return cast(
7272
CodeLocation,
7373
next(
7474
iter(workspace_context.create_request_context().get_code_location_entries().values())
7575
).code_location,
76-
).get_repository("the_repo")
76+
)
77+
78+
79+
@pytest.fixture(name="remote_repo", scope="module")
80+
def remote_repo_fixture(
81+
code_location: CodeLocation,
82+
) -> Iterator[RemoteRepository]:
83+
yield code_location.get_repository("the_repo")
7784

7885

7986
def loadable_target_origin(attribute: Optional[str] = None) -> LoadableTargetOrigin:

python_modules/dagster/dagster_tests/daemon_tests/test_backfill.py

Lines changed: 19 additions & 34 deletions
Original file line numberDiff line numberDiff line change
@@ -53,11 +53,13 @@
5353
get_asset_backfill_run_chunk_size,
5454
)
5555
from dagster._core.execution.backfill import BulkActionStatus, PartitionBackfill
56+
from dagster._core.execution.plan.resume_retry import ReexecutionStrategy
5657
from dagster._core.remote_representation import (
5758
InProcessCodeLocationOrigin,
5859
RemoteRepository,
5960
RemoteRepositoryOrigin,
6061
)
62+
from dagster._core.remote_representation.code_location import CodeLocation
6163
from dagster._core.storage.compute_log_manager import ComputeIOType
6264
from dagster._core.storage.dagster_run import (
6365
IN_PROGRESS_RUN_STATUSES,
@@ -69,6 +71,7 @@
6971
ASSET_PARTITION_RANGE_END_TAG,
7072
ASSET_PARTITION_RANGE_START_TAG,
7173
BACKFILL_ID_TAG,
74+
BACKFILL_TAGS,
7275
MAX_RETRIES_TAG,
7376
PARTITION_NAME_TAG,
7477
)
@@ -3169,9 +3172,9 @@ def test_asset_backfill_retries_make_downstreams_runnable(
31693172
def test_run_retry_not_part_of_completed_backfill(
31703173
instance: DagsterInstance,
31713174
workspace_context: WorkspaceProcessContext,
3175+
code_location: CodeLocation,
31723176
remote_repo: RemoteRepository,
31733177
):
3174-
del remote_repo
31753178
backfill_id = "run_retries_backfill"
31763179
partition_keys = static_partitions.get_partition_keys()
31773180
asset_selection = [AssetKey("foo"), AssetKey("a1"), AssetKey("bar")]
@@ -3215,47 +3218,29 @@ def test_run_retry_not_part_of_completed_backfill(
32153218
assert backfill.status == BulkActionStatus.COMPLETED_SUCCESS
32163219

32173220
# manual retry of a run
3218-
instance.create_reexecuted_run()
32193221

32203222
# simulate a retry of a run
32213223
run_to_retry = instance.get_runs()[0]
3222-
retried_run = create_run_for_test(
3223-
instance=instance,
3224-
job_name=run_to_retry.job_name,
3225-
tags=run_to_retry.tags,
3226-
root_run_id=run_to_retry.run_id,
3227-
parent_run_id=run_to_retry.run_id,
3224+
remote_job = remote_repo.get_full_job(run_to_retry.job_name)
3225+
retried_run = instance.create_reexecuted_run(
3226+
parent_run=run_to_retry,
3227+
code_location=code_location,
3228+
remote_job=remote_job,
3229+
strategy=ReexecutionStrategy.ALL_STEPS,
32283230
)
32293231

3230-
# since there is a run in progress, the backfill should not be marked as complete, even though
3231-
# all targeted asset partitions have a completed state
3232+
for tag in BACKFILL_TAGS:
3233+
assert tag not in retried_run.tags.keys()
3234+
3235+
# Since the backfill is alerady complete, it should not be processed by the backfill daemon and
3236+
# should remain in a completed state
32323237
list(execute_backfill_iteration(workspace_context, get_default_daemon_logger("BackfillDaemon")))
32333238
backfill = instance.get_backfill(backfill_id)
32343239
assert backfill
3235-
assert backfill.asset_backfill_data
3236-
assert backfill.asset_backfill_data.all_targeted_partitions_have_materialization_status()
32373240
assert backfill.status == BulkActionStatus.REQUESTED
32383241

3239-
# manually mark the run as successful to show that the backfill will be marked as complete
3240-
# since there are no in progress runs
3241-
instance.handle_new_event(
3242-
EventLogEntry(
3243-
error_info=None,
3244-
level="debug",
3245-
user_message="",
3246-
run_id=retried_run.run_id,
3247-
timestamp=time.time(),
3248-
dagster_event=DagsterEvent(
3249-
event_type_value=DagsterEventType.RUN_SUCCESS.value,
3250-
job_name=retried_run.job_name,
3251-
),
3252-
)
3253-
)
3254-
3255-
retried_run = instance.get_runs(filters=RunsFilter(run_ids=[retried_run.run_id]))[0]
3256-
assert retried_run.status == DagsterRunStatus.SUCCESS
3242+
wait_for_all_runs_to_finish(instance, timeout=30)
32573243

3258-
list(execute_backfill_iteration(workspace_context, get_default_daemon_logger("BackfillDaemon")))
3259-
backfill = instance.get_backfill(backfill_id)
3260-
assert backfill
3261-
assert backfill.status == BulkActionStatus.COMPLETED_SUCCESS
3244+
assert retried_run.run_id not in [
3245+
r.run_id for r in instance.get_runs(filters=RunsFilter.for_backfill(backfill_id))
3246+
]

0 commit comments

Comments
 (0)