Working with TaskFlow

This tutorial builds on the regular Airflow Tutorial and focuses specifically on writing data pipelines using the TaskFlow API paradigm which is introduced as part of Airflow 2.0 and contrasts this with DAGs written using the traditional paradigm.

The data pipeline chosen here is a simple pattern with three separate Extract, Transform, and Load tasks.

Example “TaskFlow API” Pipeline

Here is a very simple pipeline using the TaskFlow API paradigm. A more detailed explanation is given below.

airflow/example_dags/tutorial_taskflow_api.py[source]

  1. import json
  2. import pendulum
  3. from airflow.decorators import dag, task
  4. @dag(
  5. schedule=None,
  6. start_date=pendulum.datetime(2021, 1, 1, tz="UTC"),
  7. catchup=False,
  8. tags=["example"],
  9. )
  10. def tutorial_taskflow_api():
  11. """
  12. ### TaskFlow API Tutorial Documentation
  13. This is a simple data pipeline example which demonstrates the use of
  14. the TaskFlow API using three simple tasks for Extract, Transform, and Load.
  15. Documentation that goes along with the Airflow TaskFlow API tutorial is
  16. located
  17. [here](https://airflow.apache.org/docs/apache-airflow/stable/tutorial_taskflow_api.html)
  18. """
  19. @task()
  20. def extract():
  21. """
  22. #### Extract task
  23. A simple Extract task to get data ready for the rest of the data
  24. pipeline. In this case, getting data is simulated by reading from a
  25. hardcoded JSON string.
  26. """
  27. data_string = '{"1001": 301.27, "1002": 433.21, "1003": 502.22}'
  28. order_data_dict = json.loads(data_string)
  29. return order_data_dict
  30. @task(multiple_outputs=True)
  31. def transform(order_data_dict: dict):
  32. """
  33. #### Transform task
  34. A simple Transform task which takes in the collection of order data and
  35. computes the total order value.
  36. """
  37. total_order_value = 0
  38. for value in order_data_dict.values():
  39. total_order_value += value
  40. return {"total_order_value": total_order_value}
  41. @task()
  42. def load(total_order_value: float):
  43. """
  44. #### Load task
  45. A simple Load task which takes in the result of the Transform task and
  46. instead of saving it to end user review, just prints it out.
  47. """
  48. print(f"Total order value is: {total_order_value:.2f}")
  49. order_data = extract()
  50. order_summary = transform(order_data)
  51. load(order_summary["total_order_value"])
  52. tutorial_taskflow_api()

It’s a DAG definition file

If this is the first DAG file you are looking at, please note that this Python script is interpreted by Airflow and is a configuration file for your data pipeline. For a complete introduction to DAG files, please look at the core fundamentals tutorial which covers DAG structure and definitions extensively.

Instantiate a DAG

We are creating a DAG which is the collection of our tasks with dependencies between the tasks. This is a very simple definition, since we just want the DAG to be run when we set this up with Airflow, without any retries or complex scheduling. In this example, please notice that we are creating this DAG using the @dag decorator as shown below, with the Python function name acting as the DAG identifier.

airflow/example_dags/tutorial_taskflow_api.py[source]

  1. @dag(
  2. schedule=None,
  3. start_date=pendulum.datetime(2021, 1, 1, tz="UTC"),
  4. catchup=False,
  5. tags=["example"],
  6. )
  7. def tutorial_taskflow_api():
  8. """
  9. ### TaskFlow API Tutorial Documentation
  10. This is a simple data pipeline example which demonstrates the use of
  11. the TaskFlow API using three simple tasks for Extract, Transform, and Load.
  12. Documentation that goes along with the Airflow TaskFlow API tutorial is
  13. located
  14. [here](https://airflow.apache.org/docs/apache-airflow/stable/tutorial_taskflow_api.html)
  15. """

Now to actually enable this to be run as a DAG, we invoke the Python function tutorial_taskflow_api set up using the @dag decorator earlier, as shown below.

airflow/example_dags/tutorial_taskflow_api.py[source]

  1. tutorial_taskflow_api()

Changed in version 2.4: It’s no longer required to “register” the DAG into a global variable for Airflow to be able to detect the dag if that DAG is used inside a with block, or if it is the result of a @dag decorated function.

Tasks

In this data pipeline, tasks are created based on Python functions using the @task decorator as shown below. The function name acts as a unique identifier for the task.

airflow/example_dags/tutorial_taskflow_api.py[source]

  1. @task()
  2. def extract():
  3. """
  4. #### Extract task
  5. A simple Extract task to get data ready for the rest of the data
  6. pipeline. In this case, getting data is simulated by reading from a
  7. hardcoded JSON string.
  8. """
  9. data_string = '{"1001": 301.27, "1002": 433.21, "1003": 502.22}'
  10. order_data_dict = json.loads(data_string)
  11. return order_data_dict

The returned value, which in this case is a dictionary, will be made available for use in later tasks.

The Transform and Load tasks are created in the same manner as the Extract task shown above.

Main flow of the DAG

Now that we have the Extract, Transform, and Load tasks defined based on the Python functions, we can move to the main part of the DAG.

airflow/example_dags/tutorial_taskflow_api.py[source]

  1. order_data = extract()
  2. order_summary = transform(order_data)
  3. load(order_summary["total_order_value"])

That’s it, we are done! We have invoked the Extract task, obtained the order data from there and sent it over to the Transform task for summarization, and then invoked the Load task with the summarized data. The dependencies between the tasks and the passing of data between these tasks which could be running on different workers on different nodes on the network is all handled by Airflow.

Now to actually enable this to be run as a DAG, we invoke the Python function tutorial_taskflow_api set up using the @dag decorator earlier, as shown below.

airflow/example_dags/tutorial_taskflow_api.py[source]

  1. tutorial_taskflow_api()

But how?

For experienced Airflow DAG authors, this is startlingly simple! Let’s contrast this with how this DAG had to be written before Airflow 2.0 below:

airflow/example_dags/tutorial_dag.py[source]

  1. import json
  2. from textwrap import dedent
  3. import pendulum
  4. # The DAG object; we'll need this to instantiate a DAG
  5. from airflow import DAG
  6. # Operators; we need this to operate!
  7. from airflow.operators.python import PythonOperator
  8. with DAG(
  9. "tutorial_dag",
  10. # These args will get passed on to each operator
  11. # You can override them on a per-task basis during operator initialization
  12. default_args={"retries": 2},
  13. description="DAG tutorial",
  14. schedule=None,
  15. start_date=pendulum.datetime(2021, 1, 1, tz="UTC"),
  16. catchup=False,
  17. tags=["example"],
  18. ) as dag:
  19. dag.doc_md = __doc__
  20. def extract(**kwargs):
  21. ti = kwargs["ti"]
  22. data_string = '{"1001": 301.27, "1002": 433.21, "1003": 502.22}'
  23. ti.xcom_push("order_data", data_string)
  24. def transform(**kwargs):
  25. ti = kwargs["ti"]
  26. extract_data_string = ti.xcom_pull(task_ids="extract", key="order_data")
  27. order_data = json.loads(extract_data_string)
  28. total_order_value = 0
  29. for value in order_data.values():
  30. total_order_value += value
  31. total_value = {"total_order_value": total_order_value}
  32. total_value_json_string = json.dumps(total_value)
  33. ti.xcom_push("total_order_value", total_value_json_string)
  34. def load(**kwargs):
  35. ti = kwargs["ti"]
  36. total_value_string = ti.xcom_pull(task_ids="transform", key="total_order_value")
  37. total_order_value = json.loads(total_value_string)
  38. print(total_order_value)
  39. extract_task = PythonOperator(
  40. task_id="extract",
  41. python_callable=extract,
  42. )
  43. extract_task.doc_md = dedent(
  44. """\
  45. #### Extract task
  46. A simple Extract task to get data ready for the rest of the data pipeline.
  47. In this case, getting data is simulated by reading from a hardcoded JSON string.
  48. This data is then put into xcom, so that it can be processed by the next task.
  49. """
  50. )
  51. transform_task = PythonOperator(
  52. task_id="transform",
  53. python_callable=transform,
  54. )
  55. transform_task.doc_md = dedent(
  56. """\
  57. #### Transform task
  58. A simple Transform task which takes in the collection of order data from xcom
  59. and computes the total order value.
  60. This computed value is then put into xcom, so that it can be processed by the next task.
  61. """
  62. )
  63. load_task = PythonOperator(
  64. task_id="load",
  65. python_callable=load,
  66. )
  67. load_task.doc_md = dedent(
  68. """\
  69. #### Load task
  70. A simple Load task which takes in the result of the Transform task, by reading it
  71. from xcom and instead of saving it to end user review, just prints it out.
  72. """
  73. )
  74. extract_task >> transform_task >> load_task

All of the processing shown above is being done in the new Airflow 2.0 DAG as well, but it is all abstracted from the DAG developer.

Let’s examine this in detail by looking at the Transform task in isolation since it is in the middle of the data pipeline. In Airflow 1.x, this task is defined as shown below:

airflow/example_dags/tutorial_dag.py[source]

  1. def transform(**kwargs):
  2. ti = kwargs["ti"]
  3. extract_data_string = ti.xcom_pull(task_ids="extract", key="order_data")
  4. order_data = json.loads(extract_data_string)
  5. total_order_value = 0
  6. for value in order_data.values():
  7. total_order_value += value
  8. total_value = {"total_order_value": total_order_value}
  9. total_value_json_string = json.dumps(total_value)
  10. ti.xcom_push("total_order_value", total_value_json_string)

As we see here, the data being processed in the Transform function is passed to it using XCom variables. In turn, the summarized data from the Transform function is also placed into another XCom variable which will then be used by the Load task.

Contrasting that with TaskFlow API in Airflow 2.0 as shown below.

airflow/example_dags/tutorial_taskflow_api.py[source]

  1. @task(multiple_outputs=True)
  2. def transform(order_data_dict: dict):
  3. """
  4. #### Transform task
  5. A simple Transform task which takes in the collection of order data and
  6. computes the total order value.
  7. """
  8. total_order_value = 0
  9. for value in order_data_dict.values():
  10. total_order_value += value
  11. return {"total_order_value": total_order_value}

All of the XCom usage for data passing between these tasks is abstracted away from the DAG author in Airflow 2.0. However, XCom variables are used behind the scenes and can be viewed using the Airflow UI as necessary for debugging or DAG monitoring.

Similarly, task dependencies are automatically generated within TaskFlows based on the functional invocation of tasks. In Airflow 1.x, tasks had to be explicitly created and dependencies specified as shown below.

airflow/example_dags/tutorial_dag.py[source]

  1. extract_task = PythonOperator(
  2. task_id="extract",
  3. python_callable=extract,
  4. )
  5. extract_task.doc_md = dedent(
  6. """\
  7. #### Extract task
  8. A simple Extract task to get data ready for the rest of the data pipeline.
  9. In this case, getting data is simulated by reading from a hardcoded JSON string.
  10. This data is then put into xcom, so that it can be processed by the next task.
  11. """
  12. )
  13. transform_task = PythonOperator(
  14. task_id="transform",
  15. python_callable=transform,
  16. )
  17. transform_task.doc_md = dedent(
  18. """\
  19. #### Transform task
  20. A simple Transform task which takes in the collection of order data from xcom
  21. and computes the total order value.
  22. This computed value is then put into xcom, so that it can be processed by the next task.
  23. """
  24. )
  25. load_task = PythonOperator(
  26. task_id="load",
  27. python_callable=load,
  28. )
  29. load_task.doc_md = dedent(
  30. """\
  31. #### Load task
  32. A simple Load task which takes in the result of the Transform task, by reading it
  33. from xcom and instead of saving it to end user review, just prints it out.
  34. """
  35. )
  36. extract_task >> transform_task >> load_task

In contrast, with the TaskFlow API in Airflow 2.0, the invocation itself automatically generates the dependencies as shown below.

airflow/example_dags/tutorial_taskflow_api.py[source]

  1. order_data = extract()
  2. order_summary = transform(order_data)
  3. load(order_summary["total_order_value"])

Reusing a decorated task

Decorated tasks are flexible. You can reuse a decorated task in multiple DAGs, overriding the task parameters such as the task_id, queue, pool, etc.

Below is an example of how you can reuse a decorated task in multiple DAGs:

  1. from airflow.decorators import task, dag
  2. from datetime import datetime
  3. @task
  4. def add_task(x, y):
  5. print(f"Task args: x={x}, y={y}")
  6. return x + y
  7. @dag(start_date=datetime(2022, 1, 1))
  8. def mydag():
  9. start = add_task.override(task_id="start")(1, 2)
  10. for i in range(3):
  11. start >> add_task.override(task_id=f"add_start_{i}")(start, i)
  12. @dag(start_date=datetime(2022, 1, 1))
  13. def mydag2():
  14. start = add_task(1, 2)
  15. for i in range(3):
  16. start >> add_task.override(task_id=f"new_add_task_{i}")(start, i)
  17. first_dag = mydag()
  18. second_dag = mydag2()

You can also import the above add_task and use it in another DAG file. Suppose the add_task code lives in a file called common.py. You can do this:

  1. from common import add_task
  2. from airflow.decorators import dag
  3. from datetime import datetime
  4. @dag(start_date=datetime(2022, 1, 1))
  5. def use_add_task():
  6. start = add_task.override(priority_weight=3)(1, 2)
  7. for i in range(3):
  8. start >> add_task.override(task_id=f"new_add_task_{i}", retries=4)(start, i)
  9. created_dag = use_add_task()

Using the TaskFlow API with complex/conflicting Python dependencies

If you have tasks that require complex or conflicting requirements then you will have the ability to use the TaskFlow API with either Python virtual environment (since 2.0.2), Docker container (since 2.2.0), ExternalPythonOperator (since 2.4.0) or KubernetesPodOperator (since 2.4.0).

This functionality allows a much more comprehensive range of use-cases for the TaskFlow API, as you are not limited to the packages and system libraries of the Airflow worker. For all cases of the decorated functions described below, you have to make sure the functions are serializable and that they only use local imports for additional dependencies you use. Those imported additional libraries must be available in the target environment - they do not need to be available in the main Airflow environment.

Which of the operators you should use, depend on several factors:

  • whether you are running Airflow with access to Docker engine or Kubernetes

  • whether you can afford an overhead to dynamically create a virtual environment with the new dependencies

  • whether you can deploy a pre-existing, immutable Python environment for all Airflow components.

These options should allow for far greater flexibility for users who wish to keep their workflows simpler and more Pythonic - and allow you to keep complete logic of your DAG in the DAG itself.

You can also get more context about the approach of managing conflicting dependencies, including more detailed explanation on boundaries and consequences of each of the options in Best practices for handling conflicting/complex Python dependencies

Virtualenv created dynamically for each task

The simplest approach is to create dynamically (every time a task is run) a separate virtual environment on the same machine, you can use the @task.virtualenv decorator. The decorator allows you to create dynamically a new virtualenv with custom libraries and even a different Python version to run your function.

Example (dynamically created virtualenv):

airflow/example_dags/example_python_operator.py[source]

  1. @task.virtualenv(
  2. task_id="virtualenv_python", requirements=["colorama==0.4.0"], system_site_packages=False
  3. )
  4. def callable_virtualenv():
  5. """
  6. Example function that will be performed in a virtual environment.
  7. Importing at the module level ensures that it will not attempt to import the
  8. library before it is installed.
  9. """
  10. from time import sleep
  11. from colorama import Back, Fore, Style
  12. print(Fore.RED + "some red text")
  13. print(Back.GREEN + "and with a green background")
  14. print(Style.DIM + "and in dim text")
  15. print(Style.RESET_ALL)
  16. for _ in range(4):
  17. print(Style.DIM + "Please wait...", flush=True)
  18. sleep(1)
  19. print("Finished")
  20. virtualenv_task = callable_virtualenv()

Using Python environment with pre-installed dependencies

A bit more involved @task.external_python decorator allows you to run an Airflow task in pre-defined, immutable virtualenv (or Python binary installed at system level without virtualenv). This virtualenv or system python can also have different set of custom libraries installed and must be made available in all workers that can execute the tasks in the same location.

Example with @task.external_python (using immutable, pre-existing virtualenv):

airflow/example_dags/example_python_operator.py[source]

  1. @task.external_python(task_id="external_python", python=PATH_TO_PYTHON_BINARY)
  2. def callable_external_python():
  3. """
  4. Example function that will be performed in a virtual environment.
  5. Importing at the module level ensures that it will not attempt to import the
  6. library before it is installed.
  7. """
  8. import sys
  9. from time import sleep
  10. print(f"Running task via {sys.executable}")
  11. print("Sleeping")
  12. for _ in range(4):
  13. print("Please wait...", flush=True)
  14. sleep(1)
  15. print("Finished")
  16. external_python_task = callable_external_python()

Dependency separation using Docker Operator

If your Airflow workers have access to a docker engine, you can instead use a DockerOperator and add any needed arguments to correctly run the task. Please note that the docker image must have a working Python installed and take in a bash command as the command argument.

It is worth noting that the Python source code (extracted from the decorated function) and any callable args are sent to the container via (encoded and pickled) environment variables so the length of these is not boundless (the exact limit depends on system settings).

Below is an example of using the @task.docker decorator to run a Python task.

tests/system/providers/docker/example_taskflow_api_docker_virtualenv.py[source]

  1. @task.docker(image="python:3.9-slim-bullseye", multiple_outputs=True)
  2. def transform(order_data_dict: dict):
  3. """
  4. #### Transform task
  5. A simple Transform task which takes in the collection of order data and
  6. computes the total order value.
  7. """
  8. total_order_value = 0
  9. for value in order_data_dict.values():
  10. total_order_value += value
  11. return {"total_order_value": total_order_value}

Notes on using the operator:

Note

Using @task.docker decorator in one of the earlier Airflow versions

Since @task.docker decorator is available in the docker provider, you might be tempted to use it in Airflow version before 2.2, but this is not going to work. You will get this error if you try:

  1. AttributeError: '_TaskDecorator' object has no attribute 'docker'

You should upgrade to Airflow 2.2 or above in order to use it.

Dependency separation using Kubernetes Pod Operator

If your Airflow workers have access to Kubernetes, you can instead use a KubernetesPodOperator and add any needed arguments to correctly run the task.

Below is an example of using the @task.kubernetes decorator to run a Python task.

tests/system/providers/cncf/kubernetes/example_kubernetes_decorator.py[source]

  1. @task.kubernetes(
  2. image="python:3.8-slim-buster",
  3. name="k8s_test",
  4. namespace="default",
  5. in_cluster=False,
  6. config_file="/path/to/.kube/config",
  7. )
  8. def execute_in_k8s_pod():
  9. import time
  10. print("Hello from k8s pod")
  11. time.sleep(2)
  12. @task.kubernetes(image="python:3.8-slim-buster", namespace="default", in_cluster=False)
  13. def print_pattern():
  14. n = 5
  15. for i in range(0, n):
  16. # inner loop to handle number of columns
  17. # values changing acc. to outer loop
  18. for j in range(0, i + 1):
  19. # printing stars
  20. print("* ", end="")
  21. # ending line after each row
  22. print("\r")
  23. execute_in_k8s_pod_instance = execute_in_k8s_pod()
  24. print_pattern_instance = print_pattern()
  25. execute_in_k8s_pod_instance >> print_pattern_instance

Notes on using the operator:

Note

Using @task.kubernetes decorator in one of the earlier Airflow versions

Since @task.kubernetes decorator is available in the docker provider, you might be tempted to use it in Airflow version before 2.4, but this is not going to work. You will get this error if you try:

  1. AttributeError: '_TaskDecorator' object has no attribute 'kubernetes'

You should upgrade to Airflow 2.4 or above in order to use it.

Using the TaskFlow API with Sensor operators

You can apply the @task.sensor decorator to convert a regular Python function to an instance of the BaseSensorOperator class. The Python function implements the poke logic and returns an instance of the PokeReturnValue class as the poke() method in the BaseSensorOperator does. The PokeReturnValue is a new feature in Airflow 2.3 that allows a sensor operator to push an XCom value as described in section “Having sensors return XCOM values” of Community Providers.

Alternatively in cases where the sensor doesn’t need to push XCOM values: both poke() and the wrapped function can return a boolean-like value where True designates the sensor’s operation as complete and False designates the sensor’s operation as incomplete.

airflow/example_dags/example_sensor_decorator.py[source]

  1. import pendulum
  2. from airflow.decorators import dag, task
  3. from airflow.sensors.base import PokeReturnValue
  4. @dag(
  5. schedule=None,
  6. start_date=pendulum.datetime(2021, 1, 1, tz="UTC"),
  7. catchup=False,
  8. tags=["example"],
  9. )
  10. def example_sensor_decorator():
  11. # Using a sensor operator to wait for the upstream data to be ready.
  12. @task.sensor(poke_interval=60, timeout=3600, mode="reschedule")
  13. def wait_for_upstream() -> PokeReturnValue:
  14. return PokeReturnValue(is_done=True, xcom_value="xcom_value")
  15. @task
  16. def dummy_operator() -> None:
  17. pass
  18. wait_for_upstream() >> dummy_operator()
  19. tutorial_etl_dag = example_sensor_decorator()

Multiple outputs inference

Tasks can also infer multiple outputs by using dict Python typing.

  1. @task
  2. def identity_dict(x: int, y: int) -> dict[str, int]:
  3. return {"x": x, "y": y}

By using the typing Dict for the function return type, the multiple_outputs parameter is automatically set to true.

Note, If you manually set the multiple_outputs parameter the inference is disabled and the parameter value is used.

Adding dependencies between decorated and traditional tasks

The above tutorial shows how to create dependencies between TaskFlow functions. However, dependencies can also be set between traditional tasks (such as BashOperator or FileSensor) and TaskFlow functions.

Building this dependency is shown in the code below:

  1. @task()
  2. def extract_from_file():
  3. """
  4. #### Extract from file task
  5. A simple Extract task to get data ready for the rest of the data
  6. pipeline, by reading the data from a file into a pandas dataframe
  7. """
  8. order_data_file = "/tmp/order_data.csv"
  9. order_data_df = pd.read_csv(order_data_file)
  10. file_task = FileSensor(task_id="check_file", filepath="/tmp/order_data.csv")
  11. order_data = extract_from_file()
  12. file_task >> order_data

In the above code block, a new TaskFlow function is defined as extract_from_file which reads the data from a known file location. In the main DAG, a new FileSensor task is defined to check for this file. Please note that this is a Sensor task which waits for the file. Finally, a dependency between this Sensor task and the TaskFlow function is specified.

Consuming XComs between decorated and traditional tasks

As noted above, the TaskFlow API allows XComs to be consumed or passed between tasks in a manner that is abstracted away from the DAG author. This section dives further into detailed examples of how this is possible not only between TaskFlow functions but between both TaskFlow functions and traditional tasks.

You may find it necessary to consume an XCom from traditional tasks, either pushed within the task’s execution or via its return value, as an input into downstream tasks. You can access the pushed XCom (also known as an XComArg) by utilizing the .output property exposed for all operators.

By default, using the .output property to retrieve an XCom result is the equivalent of:

  1. task_instance.xcom_pull(task_ids="my_task_id", key="return_value")

To retrieve an XCom result for a key other than return_value, you can use:

  1. my_op = MyOperator(...)
  2. my_op_output = my_op.output["some_other_xcom_key"]
  3. # OR
  4. my_op_output = my_op.output.get("some_other_xcom_key")

Note

Using the .output property as an input to another task is supported only for operator parameters listed as a template_field.

In the code example below, a SimpleHttpOperator result is captured via XComs. This XCom result, which is the task output, is then passed to a TaskFlow function which parses the response as JSON.

  1. get_api_results_task = SimpleHttpOperator(
  2. task_id="get_api_results",
  3. endpoint="/api/query",
  4. do_xcom_push=True,
  5. http_conn_id="http",
  6. )
  7. @task
  8. def parse_results(api_results):
  9. return json.loads(api_results)
  10. parsed_results = parse_results(api_results=get_api_results_task.output)

The reverse can also be done: passing the output of a TaskFlow function as an input to a traditional task.

  1. @task(retries=3)
  2. def create_queue():
  3. """This is a Python function that creates an SQS queue"""
  4. hook = SqsHook()
  5. result = hook.create_queue(queue_name="sample-queue")
  6. return result["QueueUrl"]
  7. sqs_queue = create_queue()
  8. publish_to_queue = SqsPublishOperator(
  9. task_id="publish_to_queue",
  10. sqs_queue=sqs_queue,
  11. message_content="{{ task_instance }}-{{ execution_date }}",
  12. message_attributes=None,
  13. delay_seconds=0,
  14. )

Take note in the code example above, the output from the create_queue TaskFlow function, the URL of a newly-created Amazon SQS Queue, is then passed to a SqsPublishOperator task as the sqs_queue arg.

Finally, not only can you use traditional operator outputs as inputs for TaskFlow functions, but also as inputs to other traditional operators. In the example below, the output from the SalesforceToS3Operator task (which is an S3 URI for a destination file location) is used an input for the S3CopyObjectOperator task to copy the same file to a date-partitioned storage location in S3 for long-term storage in a data lake.

  1. BASE_PATH = "salesforce/customers"
  2. FILE_NAME = "customer_daily_extract_{{ ds_nodash }}.csv"
  3. upload_salesforce_data_to_s3_landing = SalesforceToS3Operator(
  4. task_id="upload_salesforce_data_to_s3",
  5. salesforce_query="SELECT Id, Name, Company, Phone, Email, LastModifiedDate, IsActive FROM Customers",
  6. s3_bucket_name="landing-bucket",
  7. s3_key=f"{BASE_PATH}/{FILE_NAME}",
  8. salesforce_conn_id="salesforce",
  9. aws_conn_id="s3",
  10. replace=True,
  11. )
  12. store_to_s3_data_lake = S3CopyObjectOperator(
  13. task_id="store_to_s3_data_lake",
  14. aws_conn_id="s3",
  15. source_bucket_key=upload_salesforce_data_to_s3_landing.output,
  16. dest_bucket_name="data_lake",
  17. dest_bucket_key=f"""{BASE_PATH}/{"{{ execution_date.strftime('%Y/%m/%d') }}"}/{FILE_NAME}""",
  18. )

Accessing context variables in decorated tasks

When running your callable, Airflow will pass a set of keyword arguments that can be used in your function. This set of kwargs correspond exactly to what you can use in your Jinja templates. For this to work, you need to define **kwargs in your function header, or you can add directly the keyword arguments you would like to get - for example with the below code your callable will get the values of ti and next_ds context variables. Note that when explicit keyword arguments are used, they must be made optional in the function header to avoid TypeError exceptions during DAG parsing as these values are not available until task execution.

With explicit arguments:

  1. @task
  2. def my_python_callable(ti=None, next_ds=None):
  3. pass

With kwargs:

  1. @task
  2. def my_python_callable(**kwargs):
  3. ti = kwargs["ti"]
  4. next_ds = kwargs["next_ds"]

Also, sometimes you might want to access the context somewhere deep in the stack, but you do not want to pass the context variables from the task callable. You can still access execution context via the get_current_context method.

  1. from airflow.operators.python import get_current_context
  2. def some_function_in_your_library():
  3. context = get_current_context()
  4. ti = context["ti"]

Current context is accessible only during the task execution. The context is not accessible during pre_execute or post_execute. Calling this method outside execution context will raise an error.

What’s Next?

You have seen how simple it is to write DAGs using the TaskFlow API paradigm within Airflow 2.0. Here are a few steps you might want to take next:

See also