|
26 | 26 | import pickle
|
27 | 27 | import signal
|
28 | 28 | import sys
|
| 29 | +import time |
29 | 30 | import urllib
|
30 | 31 | from traceback import format_exception
|
31 | 32 | from typing import cast
|
|
34 | 35 | from uuid import uuid4
|
35 | 36 |
|
36 | 37 | import pendulum
|
| 38 | +import psutil |
37 | 39 | import pytest
|
38 | 40 | import time_machine
|
39 | 41 | from sqlalchemy import select
|
|
83 | 85 | from airflow.sensors.base import BaseSensorOperator
|
84 | 86 | from airflow.sensors.python import PythonSensor
|
85 | 87 | from airflow.serialization.serialized_objects import SerializedBaseOperator, SerializedDAG
|
86 |
| -from airflow.settings import TIMEZONE, TracebackSessionForTests |
| 88 | +from airflow.settings import TIMEZONE, TracebackSessionForTests, reconfigure_orm |
87 | 89 | from airflow.stats import Stats
|
88 | 90 | from airflow.ti_deps.dep_context import DepContext
|
89 | 91 | from airflow.ti_deps.dependencies_deps import REQUEUEABLE_DEPS, RUNNING_DEPS
|
@@ -3587,6 +3589,43 @@ def test_handle_failure(self, create_dummy_dag, session=None):
|
3587 | 3589 | assert "task_instance" in context_arg_3
|
3588 | 3590 | mock_on_retry_3.assert_not_called()
|
3589 | 3591 |
|
| 3592 | + @provide_session |
| 3593 | + def test_handle_failure_does_not_push_stale_dagrun_model(self, dag_maker, create_dummy_dag, session=None): |
| 3594 | + session = settings.Session() |
| 3595 | + with dag_maker(): |
| 3596 | + |
| 3597 | + def method(): ... |
| 3598 | + |
| 3599 | + task = PythonOperator(task_id="mytask", python_callable=method) |
| 3600 | + dr = dag_maker.create_dagrun() |
| 3601 | + ti = dr.get_task_instance(task.task_id) |
| 3602 | + ti.state = State.RUNNING |
| 3603 | + |
| 3604 | + assert dr.state == DagRunState.RUNNING |
| 3605 | + |
| 3606 | + session.merge(ti) |
| 3607 | + session.flush() |
| 3608 | + session.commit() |
| 3609 | + |
| 3610 | + pid = os.fork() |
| 3611 | + if pid: |
| 3612 | + process = psutil.Process(pid) |
| 3613 | + time.sleep(1) |
| 3614 | + |
| 3615 | + dr.state = DagRunState.SUCCESS |
| 3616 | + session.merge(dr) |
| 3617 | + session.flush() |
| 3618 | + session.commit() |
| 3619 | + process.wait(timeout=7) |
| 3620 | + else: |
| 3621 | + reconfigure_orm(disable_connection_pool=True) |
| 3622 | + time.sleep(2) |
| 3623 | + ti.handle_failure("should not update related models") |
| 3624 | + os._exit(0) |
| 3625 | + |
| 3626 | + dr.refresh_from_db() |
| 3627 | + assert dr.state == DagRunState.SUCCESS |
| 3628 | + |
3590 | 3629 | @pytest.mark.skip_if_database_isolation_mode # Does not work in db isolation mode
|
3591 | 3630 | def test_handle_failure_updates_queued_task_updates_state(self, dag_maker):
|
3592 | 3631 | session = settings.Session()
|
|
0 commit comments