Re: [PR] Bugfix: Move rendering of `map_index_template` so it renders for failed tasks as long as it was defined before the point of failure [airflow]

2024-05-30 Thread via GitHub


TJaniF commented on PR #38902:
URL: https://github.com/apache/airflow/pull/38902#issuecomment-2139219797

   @rightx2 Thanks for the added explanation. I understand what you mean now 
and can reproduce it, marking as success (or as failed) will lead to an integer 
index even if the code to define the custom map index has already run. Thanks 
for flagging this!
   
   @RNHTTR I think it would be nice if it worked for marking tasks the same way 
as for failed tasks with attempting to render the custom map index template 
even if interrupted. :) 
   If you agree I can open an issue and take a stab at it, though I can't 
promise that will happen soonish, am working on some other things rn  
   
   


-- 
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: commits-unsubscr...@airflow.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



Re: [PR] Bugfix: Move rendering of `map_index_template` so it renders for failed tasks as long as it was defined before the point of failure [airflow]

2024-05-28 Thread via GitHub


rightx2 commented on PR #38902:
URL: https://github.com/apache/airflow/pull/38902#issuecomment-2136345896

   @TJaniF 
   
   I think there was a mistake in my experiment. I believe I can clarify it now.
   
   1. For whole newly added dynamic tasks:
  - When I start running dyanmic tasks for the first time and stop them 
immediately (marking them as success or failed, before they finish), all 
`map_index` values are integers.
   
   2. For newly added individual tasks within pre-existing dynamic tasks:
  - Similarly, if I clear the tasks this time and stop them, only the 
`map_index` of the newly added individual task is an integer.


-- 
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: commits-unsubscr...@airflow.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



Re: [PR] Bugfix: Move rendering of `map_index_template` so it renders for failed tasks as long as it was defined before the point of failure [airflow]

2024-05-28 Thread via GitHub


TJaniF commented on PR #38902:
URL: https://github.com/apache/airflow/pull/38902#issuecomment-2134635772

   @rightx2 That is interesting. I just tried that and for me the map index 
values from the first run "stick", but maybe I misunderstood what you are doing 
:) 
   
   This is what I tried:
   
   ![2024-05-28_10-24-49 
(1)](https://github.com/apache/airflow/assets/90063506/1b79fa05-49a5-4c29-985e-56c0769e)
   
   This is the DAG:
   
   ```python
   from airflow.decorators import dag, task
   
   @dag(
   start_date=None,
   schedule=None,
   catchup=False,
   )
   def mapping_test():
   @task(
   map_index_template="{{ custom_map_index }}"
   )
   def add_one(num):
   import time
   
   if num > 10:
   time.sleep(10)
   else:
   time.sleep(2)
   
   from airflow.operators.python import get_current_context
   
   context = get_current_context()
   context["custom_map_index"] = "Input x=" + str(num)
   
   return num + 1
   
   add_one.expand(num=[1, 2, 3, 4, 5, 10, 11, 12, 13])
   
   
   mapping_test()
   ```
   
   I'm using Airflow 2.9.1 (Astro Runtime 11.3.0)
   


-- 
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: commits-unsubscr...@airflow.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



Re: [PR] Bugfix: Move rendering of `map_index_template` so it renders for failed tasks as long as it was defined before the point of failure [airflow]

2024-05-28 Thread via GitHub


rightx2 commented on PR #38902:
URL: https://github.com/apache/airflow/pull/38902#issuecomment-2134469117

   I wonder what I've experienced is same type of the problem above:
   
   1. clear dynnamic mapped tasks
   2. all dynamic mapped tasks go into state of running
   3. Force to set dag run state success (mark success dag run)
   
   In this case, all map_index values are integer, instead of given names.


-- 
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: commits-unsubscr...@airflow.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



Re: [PR] Bugfix: Move rendering of `map_index_template` so it renders for failed tasks as long as it was defined before the point of failure [airflow]

2024-04-15 Thread via GitHub


Lee-W merged PR #38902:
URL: https://github.com/apache/airflow/pull/38902


-- 
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: commits-unsubscr...@airflow.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



Re: [PR] Bugfix: Move rendering of `map_index_template` so it renders for failed tasks as long as it was defined before the point of failure [airflow]

2024-04-12 Thread via GitHub


RNHTTR commented on code in PR #38902:
URL: https://github.com/apache/airflow/pull/38902#discussion_r1562596410


##
airflow/models/taskinstance.py:
##
@@ -2714,18 +2714,29 @@ def signal_handler(signum, frame):
 previous_state=TaskInstanceState.QUEUED, task_instance=self, 
session=session
 )
 
-# Execute the task
+def _render_map_index(context: Context, *, jinja_env: 
jinja2.Environment | None) -> str | None:
+"""Render named map index if the DAG author defined 
map_index_template at the task level."""
+if jinja_env is None or (template := 
context.get("map_index_template")) is None:
+return None
+rendered_map_index = 
jinja_env.from_string(template).render(context)
+log.info("Map index rendered as %s", rendered_map_index)

Review Comment:
   Looks like it got changed to debug by now, which is fine.



-- 
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: commits-unsubscr...@airflow.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



Re: [PR] Bugfix: Move rendering of `map_index_template` so it renders for failed tasks as long as it was defined before the point of failure [airflow]

2024-04-12 Thread via GitHub


RNHTTR commented on code in PR #38902:
URL: https://github.com/apache/airflow/pull/38902#discussion_r1562594872


##
airflow/models/taskinstance.py:
##
@@ -2714,18 +2714,29 @@ def signal_handler(signum, frame):
 previous_state=TaskInstanceState.QUEUED, task_instance=self, 
session=session
 )
 
-# Execute the task
+def _render_map_index(context: Context, *, jinja_env: 
jinja2.Environment | None) -> str | None:
+"""Render named map index if the DAG author defined 
map_index_template at the task level."""
+if jinja_env is None or (template := 
context.get("map_index_template")) is None:
+return None
+rendered_map_index = 
jinja_env.from_string(template).render(context)
+log.info("Map index rendered as %s", rendered_map_index)

Review Comment:
   I think it was useful before I hooked up the UI, but it should be fine to 
remove.



-- 
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: commits-unsubscr...@airflow.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



Re: [PR] Bugfix: Move rendering of `map_index_template` so it renders for failed tasks as long as it was defined before the point of failure [airflow]

2024-04-12 Thread via GitHub


uranusjr commented on code in PR #38902:
URL: https://github.com/apache/airflow/pull/38902#discussion_r1562406985


##
airflow/models/taskinstance.py:
##
@@ -2714,18 +2714,29 @@ def signal_handler(signum, frame):
 previous_state=TaskInstanceState.QUEUED, task_instance=self, 
session=session
 )
 
-# Execute the task
+def _render_map_index(context: Context, *, jinja_env: 
jinja2.Environment | None) -> str | None:
+"""Render named map index if the DAG author defined 
map_index_template at the task level."""
+if jinja_env is None or (template := 
context.get("map_index_template")) is None:
+return None
+rendered_map_index = 
jinja_env.from_string(template).render(context)
+log.info("Map index rendered as %s", rendered_map_index)

Review Comment:
   @RNHTTR thoughts? (Since you actually used this and debugged errors with it 
in practice)



-- 
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: commits-unsubscr...@airflow.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



Re: [PR] Bugfix: Move rendering of `map_index_template` so it renders for failed tasks as long as it was defined before the point of failure [airflow]

2024-04-12 Thread via GitHub


kaxil commented on code in PR #38902:
URL: https://github.com/apache/airflow/pull/38902#discussion_r1562401551


##
airflow/models/taskinstance.py:
##
@@ -2714,18 +2714,29 @@ def signal_handler(signum, frame):
 previous_state=TaskInstanceState.QUEUED, task_instance=self, 
session=session
 )
 
-# Execute the task
+def _render_map_index(context: Context, *, jinja_env: 
jinja2.Environment | None) -> str | None:
+"""Render named map index if the DAG author defined 
map_index_template at the task level."""
+if jinja_env is None or (template := 
context.get("map_index_template")) is None:
+return None
+rendered_map_index = 
jinja_env.from_string(template).render(context)
+log.info("Map index rendered as %s", rendered_map_index)

Review Comment:
   Yeah, worth removing or changing it to debug level (if we think it could be 
useful in debugging)



-- 
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: commits-unsubscr...@airflow.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



Re: [PR] Bugfix: Move rendering of `map_index_template` so it renders for failed tasks as long as it was defined before the point of failure [airflow]

2024-04-12 Thread via GitHub


ashb commented on code in PR #38902:
URL: https://github.com/apache/airflow/pull/38902#discussion_r1562367870


##
airflow/models/taskinstance.py:
##
@@ -2714,18 +2714,29 @@ def signal_handler(signum, frame):
 previous_state=TaskInstanceState.QUEUED, task_instance=self, 
session=session
 )
 
-# Execute the task
+def _render_map_index(context: Context, *, jinja_env: 
jinja2.Environment | None) -> str | None:
+"""Render named map index if the DAG author defined 
map_index_template at the task level."""
+if jinja_env is None or (template := 
context.get("map_index_template")) is None:
+return None
+rendered_map_index = 
jinja_env.from_string(template).render(context)
+log.info("Map index rendered as %s", rendered_map_index)

Review Comment:
   Is this log useful?



-- 
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: commits-unsubscr...@airflow.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



Re: [PR] Bugfix: Move rendering of `map_index_template` so it renders for failed tasks as long as it was defined before the point of failure [airflow]

2024-04-12 Thread via GitHub


TJaniF commented on PR #38902:
URL: https://github.com/apache/airflow/pull/38902#issuecomment-2051345427

   @uranusjr thank you!


-- 
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: commits-unsubscr...@airflow.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



Re: [PR] Bugfix: Move rendering of `map_index_template` so it renders for failed tasks as long as it was defined before the point of failure [airflow]

2024-04-12 Thread via GitHub


uranusjr commented on PR #38902:
URL: https://github.com/apache/airflow/pull/38902#issuecomment-2051246536

   This should fix things…


-- 
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: commits-unsubscr...@airflow.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



Re: [PR] Bugfix: Move rendering of `map_index_template` so it renders for failed tasks as long as it was defined before the point of failure [airflow]

2024-04-11 Thread via GitHub


TJaniF commented on PR #38902:
URL: https://github.com/apache/airflow/pull/38902#issuecomment-2049927602

   > Need to fix tests
   
   Sorry silly question but for `breeze testing non-db-tests 
--parallel-test-types "Core"` which I think runs my two new tests
   
   I get:
   
   ```text
   === short test summary info 

   FAILED 
tests/utils/log/test_file_processor_handler.py::TestFileProcessorHandler::test_non_template
 - AssertionError: assert False
+  where False = ('/tmp/log_test/latest')
+where  = .islink
+  where  = os.path
   === 1 failed, 826 passed, 1950 skipped, 1 xfailed in 20.30s 

   ```
   
   which also fails on main ... so I am not sure if you mean I need to fix my 
tests or in general I need to wait for the tests to be fixed on main and then 
update the branch  


-- 
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: commits-unsubscr...@airflow.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



Re: [PR] Bugfix: Move rendering of `map_index_template` so it renders for failed tasks as long as it was defined before the point of failure [airflow]

2024-04-11 Thread via GitHub


TJaniF commented on code in PR #38902:
URL: https://github.com/apache/airflow/pull/38902#discussion_r1560866337


##
airflow/models/taskinstance.py:
##
@@ -2715,29 +2727,26 @@ def signal_handler(signum, frame):
 
 # Execute the task
 with set_current_context(context):
-result = self._execute_task(context, task_orig)
+result, rendered_map_index = self._execute_task(context, 
task_orig, jinja_env=jinja_env)

Review Comment:
   thank you! did that and tested it, works as intended the template error is 
suppressed for failed tasks where the failure happens before the index name is 
assigned. (I also like that this version does not involve touching 
`_execute_task`  )



-- 
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: commits-unsubscr...@airflow.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



Re: [PR] Bugfix: Move rendering of `map_index_template` so it renders for failed tasks as long as it was defined before the point of failure [airflow]

2024-04-11 Thread via GitHub


TJaniF commented on code in PR #38902:
URL: https://github.com/apache/airflow/pull/38902#discussion_r1560866337


##
airflow/models/taskinstance.py:
##
@@ -2715,29 +2727,26 @@ def signal_handler(signum, frame):
 
 # Execute the task
 with set_current_context(context):
-result = self._execute_task(context, task_orig)
+result, rendered_map_index = self._execute_task(context, 
task_orig, jinja_env=jinja_env)

Review Comment:
    did that and tested it, works as intended the template error is suppressed 
for failed tasks. (I also like that this version does not involve touching 
`_execute_task`  )



##
airflow/models/taskinstance.py:
##
@@ -2715,29 +2727,26 @@ def signal_handler(signum, frame):
 
 # Execute the task
 with set_current_context(context):
-result = self._execute_task(context, task_orig)
+result, rendered_map_index = self._execute_task(context, 
task_orig, jinja_env=jinja_env)

Review Comment:
   thank you! did that and tested it, works as intended the template error is 
suppressed for failed tasks. (I also like that this version does not involve 
touching `_execute_task`  )



-- 
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: commits-unsubscr...@airflow.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



Re: [PR] Bugfix: Move rendering of `map_index_template` so it renders for failed tasks as long as it was defined before the point of failure [airflow]

2024-04-10 Thread via GitHub


uranusjr commented on code in PR #38902:
URL: https://github.com/apache/airflow/pull/38902#discussion_r1560385357


##
airflow/models/taskinstance.py:
##
@@ -2715,29 +2727,26 @@ def signal_handler(signum, frame):
 
 # Execute the task
 with set_current_context(context):
-result = self._execute_task(context, task_orig)
+result, rendered_map_index = self._execute_task(context, 
task_orig, jinja_env=jinja_env)

Review Comment:
   I wonder if we should just try to render the template _regardless_ of the 
task at all. Something like
   
   ```python
   def _render_map_index(...):
   ...
   
   try:
   result = self._execute_task(context, task_orig)
   except Exception:
   # If the task failed, swallow rendering error so it doesn't mask the 
main error.
   with contextlib.suppress(jinja2.TemplateSyntaxError, 
jinja2.UndefinedError):
   _render_map_index(...)
   raise
   else:
   # If the task succeeded, render normally to let rendering error bubble 
up.
   _render_map_index(...)
   ```



-- 
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: commits-unsubscr...@airflow.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[PR] Bugfix: Move rendering of `map_index_template` so it renders for failed tasks as long as it was defined before the point of failure [airflow]

2024-04-10 Thread via GitHub


TJaniF opened a new pull request, #38902:
URL: https://github.com/apache/airflow/pull/38902

   In Airflow 2.9 `map_index_template` does not render when the task fails. 
   
   
![image](https://github.com/apache/airflow/assets/90063506/6c4b0398-7ce1-405b-add8-8c03399f76c8)
   
   I moved the rendering into the `finally` of `_execute_callable` so it always 
happens. 
   
   TaskFlow:
   https://github.com/apache/airflow/assets/90063506/5bf11e20-0553-4b2a-95f3-648232b86c6d;>
   
   Traditional operator:
   https://github.com/apache/airflow/assets/90063506/6dbbe5c0-aa9d-45e1-98e6-cbad446cc323;>
   
   
   Also attempted 2 unit tests :)
   
   Apologies if there is already a PR addressing this. I did not see one. 
   
   cc: @RNHTTR 
   
   
   ---
   **^ Add meaningful description above**
   Read the **[Pull Request 
Guidelines](https://github.com/apache/airflow/blob/main/contributing-docs/05_pull_requests.rst#pull-request-guidelines)**
 for more information.
   In case of fundamental code changes, an Airflow Improvement Proposal 
([AIP](https://cwiki.apache.org/confluence/display/AIRFLOW/Airflow+Improvement+Proposals))
 is needed.
   In case of a new dependency, check compliance with the [ASF 3rd Party 
License Policy](https://www.apache.org/legal/resolved.html#category-x).
   In case of backwards incompatible changes please leave a note in a 
newsfragment file, named `{pr_number}.significant.rst` or 
`{issue_number}.significant.rst`, in 
[newsfragments](https://github.com/apache/airflow/tree/main/newsfragments).
   


-- 
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: commits-unsubscr...@airflow.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org