This is an automated email from the ASF dual-hosted git repository.

jscheffl pushed a commit to branch main
in repository https://gitbox.apache.org/repos/asf/airflow.git


The following commit(s) were added to refs/heads/main by this push:
     new 9be3dc1860d Document and test xcom_pull run_id usage for triggered DAG 
runs (#63030)
9be3dc1860d is described below

commit 9be3dc1860d99e60c4c0f3b1ec30540ea43d91d4
Author: Aviral Garg <[email protected]>
AuthorDate: Sun Jun 14 14:40:23 2026 +0530

    Document and test xcom_pull run_id usage for triggered DAG runs (#63030)
    
    * Document and test xcom_pull run_id usage for triggered DAG runs
    
    * Address XCom docs review wording
---
 airflow-core/docs/core-concepts/xcoms.rst          | 10 ++++++++
 .../task_sdk/execution_time/test_task_runner.py    | 29 ++++++++++++++++++++++
 2 files changed, 39 insertions(+)

diff --git a/airflow-core/docs/core-concepts/xcoms.rst 
b/airflow-core/docs/core-concepts/xcoms.rst
index 9e9f8f75d86..b61ee9e1744 100644
--- a/airflow-core/docs/core-concepts/xcoms.rst
+++ b/airflow-core/docs/core-concepts/xcoms.rst
@@ -49,6 +49,16 @@ Many operators will auto-push their results into an XCom key 
called ``return_val
     # Pulls the return_value XCOM from "pushing_task"
     value = task_instance.xcom_pull(task_ids='pushing_task')
 
+If you need to pull a value from a specific DAG run (for example, a DAG run 
triggered by
+``TriggerDagRunOperator``), specify both ``dag_id`` and ``run_id`` explicitly::
+
+    trigger_run_id = task_instance.xcom_pull(task_ids="trigger_child", 
key="trigger_run_id")
+    child_value = task_instance.xcom_pull(
+        task_ids="child_task",
+        dag_id="child_dag",
+        run_id=trigger_run_id,
+    )
+
 The return_value key (default key with which XComs are pushed) is defined as a 
constant XCOM_RETURN_KEY in the :class:`~airflow.sdk.bases.xcom.BaseXCom` class 
and can be accessed as BaseXCom.XCOM_RETURN_KEY.
 
 You can also use XComs in :ref:`templates <concepts:jinja-templating>`::
diff --git a/task-sdk/tests/task_sdk/execution_time/test_task_runner.py 
b/task-sdk/tests/task_sdk/execution_time/test_task_runner.py
index a6e37fa2ea6..2e87b4139d9 100644
--- a/task-sdk/tests/task_sdk/execution_time/test_task_runner.py
+++ b/task-sdk/tests/task_sdk/execution_time/test_task_runner.py
@@ -3617,6 +3617,35 @@ class TestXComAfterTaskExecution:
         ]
         assert result == expected
 
+    def test_xcom_pull_with_explicit_dag_id_and_run_id(self, 
create_runtime_ti, mock_supervisor_comms):
+        task = BaseOperator(task_id="parent_task")
+        runtime_ti = create_runtime_ti(task=task, dag_id="parent_dag", 
run_id="parent_run")
+        value = {"child_result": "hello world"}
+        ser_value = BaseXCom.serialize_value(value)
+
+        mock_supervisor_comms.send.return_value = 
XComSequenceSliceResult(root=[ser_value])
+
+        assert (
+            runtime_ti.xcom_pull(
+                task_ids="child_task",
+                dag_id="child_dag",
+                run_id="child_run",
+            )
+            == value
+        )
+        mock_supervisor_comms.send.assert_called_once_with(
+            msg=GetXComSequenceSlice(
+                key="return_value",
+                dag_id="child_dag",
+                run_id="child_run",
+                task_id="child_task",
+                start=None,
+                stop=None,
+                step=None,
+                include_prior_dates=False,
+            ),
+        )
+
     @pytest.mark.parametrize(
         ("include_prior_dates", "expected_value"),
         [

Reply via email to