xBis7 commented on code in PR #61897:
URL: https://github.com/apache/airflow/pull/61897#discussion_r2852679479


##########
airflow-core/src/airflow/executors/base_executor.py:
##########
@@ -378,28 +391,14 @@ def trigger_tasks(self, open_slots: int) -> None:
 
             if isinstance(item, workloads.ExecuteTask) and hasattr(item, "ti"):
                 ti = item.ti
-
-                # If it's None, then the span for the current id hasn't been 
started.
-                if self.active_spans is not None and 
self.active_spans.get("ti:" + str(ti.id)) is None:
-                    if isinstance(ti, workloads.TaskInstance):
-                        parent_context = 
Trace.extract(ti.parent_context_carrier)
-                    else:
-                        parent_context = 
Trace.extract(ti.dag_run.context_carrier)
-                    # Start a new span using the context from the parent.
-                    # Attributes will be set once the task has finished so 
that all
-                    # values will be available (end_time, duration, etc.).
-
-                    span = Trace.start_child_span(
-                        span_name=f"{ti.task_id}",
-                        parent_context=parent_context,
-                        component="task",
-                        start_as_current=False,
-                    )
-                    self.active_spans.set("ti:" + str(ti.id), span)
-                    # Inject the current context into the carrier.
-                    carrier = Trace.inject()
-                    ti.context_carrier = carrier
-
+                if isinstance(ti, workloads.TaskInstance):

Review Comment:
   I don't remember exactly, but this is relevant to
   
   
https://github.com/astronomer/airflow/blob/add-spans-for-task-execution/airflow-core/src/airflow/executors/workloads.py#L74-L75
   
   The task instance isn't the same one as the db model. It's the running task 
version with access to just certain fields.
   
   Because there isn't db access, `ti.dag_run...` won't work.



##########
airflow-core/src/airflow/executors/base_executor.py:
##########
@@ -378,28 +391,14 @@ def trigger_tasks(self, open_slots: int) -> None:
 
             if isinstance(item, workloads.ExecuteTask) and hasattr(item, "ti"):
                 ti = item.ti
-
-                # If it's None, then the span for the current id hasn't been 
started.
-                if self.active_spans is not None and 
self.active_spans.get("ti:" + str(ti.id)) is None:
-                    if isinstance(ti, workloads.TaskInstance):
-                        parent_context = 
Trace.extract(ti.parent_context_carrier)
-                    else:
-                        parent_context = 
Trace.extract(ti.dag_run.context_carrier)
-                    # Start a new span using the context from the parent.
-                    # Attributes will be set once the task has finished so 
that all
-                    # values will be available (end_time, duration, etc.).
-
-                    span = Trace.start_child_span(
-                        span_name=f"{ti.task_id}",
-                        parent_context=parent_context,
-                        component="task",
-                        start_as_current=False,
-                    )
-                    self.active_spans.set("ti:" + str(ti.id), span)
-                    # Inject the current context into the carrier.
-                    carrier = Trace.inject()
-                    ti.context_carrier = carrier
-
+                if isinstance(ti, workloads.TaskInstance):

Review Comment:
   I don't remember all the details, but this is relevant to
   
   
https://github.com/astronomer/airflow/blob/add-spans-for-task-execution/airflow-core/src/airflow/executors/workloads.py#L74-L75
   
   The task instance isn't the same one as the db model. It's the running task 
version with access to just certain fields.
   
   Because there isn't db access, `ti.dag_run...` won't work.



-- 
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