seanghaeli commented on code in PR #66608:
URL: https://github.com/apache/airflow/pull/66608#discussion_r3383759988


##########
airflow-core/src/airflow/triggers/callback.py:
##########
@@ -17,21 +17,61 @@
 
 from __future__ import annotations
 
+import asyncio
 import logging
+import sys
 import traceback
 from collections.abc import AsyncIterator
+from pathlib import Path
 from typing import Any
 
-from airflow._shared.module_loading import accepts_context, import_string, 
qualname
+from airflow._shared.module_loading import accepts_context, import_string, 
load_mangled_dag_module, qualname
+from airflow.dag_processing.bundles.manager import DagBundlesManager
 from airflow.models.callback import CallbackState
 from airflow.triggers.base import BaseTrigger, TriggerEvent
+from airflow.utils.file import get_unique_dag_module_name
 
 log = logging.getLogger(__name__)
 
 PAYLOAD_STATUS_KEY = "state"
 PAYLOAD_BODY_KEY = "body"
 
 
+def _ensure_bundle_module_registered(callback_path: str) -> None:
+    """
+    Register an unusual_prefix_{hash}_{stem} module so import_string can find 
it.
+
+    The triggerer event loop doesn't run the DAG processor, so DAG files with 
mangled
+    module names must be registered manually. Walks all configured bundles to 
find the
+    file, then delegates the actual loading to _load_mangled_module in 
callback_supervisor.
+    """
+    mod_name = callback_path.split(".")[0]
+    if mod_name in sys.modules:
+        return
+
+    # unusual_prefix_{hex40}_{stem}  →  {stem}.py
+    parts = mod_name.split("_", 3)
+    if len(parts) < 4:
+        return
+    stem = parts[3]
+
+    try:
+        for bundle in DagBundlesManager().get_all_dag_bundles():

Review Comment:
   This would be in scope of the bundle versioning push in AIP-109. For now we 
acknowledge this as a gap: `RunTrigger` doesn't carry bundle identity, so any 
trigger that resumes after a deploy loads current code rather than the version 
it was queued with. Same limitation exists for all trigger types today.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: [email protected]

For queries about this service, please contact Infrastructure at:
[email protected]

Reply via email to