You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@airflow.apache.org by po...@apache.org on 2022/03/07 01:45:49 UTC

[airflow] branch main updated: Change default python executable to python3 for docker decorator (#21973)

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

potiuk 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 188ac51  Change default python executable to python3 for docker decorator (#21973)
188ac51 is described below

commit 188ac519964c6b6acf9d6ab144e7ff7e5538547c
Author: Kanthi <su...@gmail.com>
AuthorDate: Sun Mar 6 20:45:10 2022 -0500

    Change default python executable to python3 for docker decorator (#21973)
    
    Added support to pass python executable(python3) to Docker decorator.
---
 airflow/decorators/__init__.pyi                      |  2 ++
 airflow/providers/docker/decorators/docker.py        | 20 +++++++++++++-------
 .../providers/docker/example_dags/example_docker.py  |  1 -
 3 files changed, 15 insertions(+), 8 deletions(-)

diff --git a/airflow/decorators/__init__.pyi b/airflow/decorators/__init__.pyi
index 9ec8c88..de3f067 100644
--- a/airflow/decorators/__init__.pyi
+++ b/airflow/decorators/__init__.pyi
@@ -151,6 +151,7 @@ class TaskDecoratorCollection:
         *,
         multiple_outputs: Optional[bool] = None,
         use_dill: bool = False,  # Added by _DockerDecoratedOperator.
+        python_command: str = "python3",
         # 'command', 'retrieve_output', and 'retrieve_output_path' are filled by
         # _DockerDecoratedOperator.
         image: str,
@@ -192,6 +193,7 @@ class TaskDecoratorCollection:
             with index as key. Dict will unroll to xcom values with keys as XCom keys.
             Defaults to False.
         :param use_dill: Whether to use dill or pickle for serialization
+        :param python_command: Python command for executing functions, Default: python3
         :param image: Docker image from which to create the container.
             If image tag is omitted, "latest" will be used.
         :param api_version: Remote API version. Set to ``auto`` to automatically
diff --git a/airflow/providers/docker/decorators/docker.py b/airflow/providers/docker/decorators/docker.py
index ea5c695..0ddf1dd 100644
--- a/airflow/providers/docker/decorators/docker.py
+++ b/airflow/providers/docker/decorators/docker.py
@@ -34,10 +34,10 @@ if TYPE_CHECKING:
     from airflow.utils.context import Context
 
 
-def _generate_decode_command(env_var, file):
+def _generate_decode_command(env_var, file, python_command):
     # We don't need `f.close()` as the interpreter is about to exit anyway
     return (
-        f'python -c "import base64, os;'
+        f'{python_command} -c "import base64, os;'
         rf'x = base64.b64decode(os.environ[\"{env_var}\"]);'
         rf'f = open(\"{file}\", \"wb\"); f.write(x);"'
     )
@@ -71,14 +71,24 @@ class _DockerDecoratedOperator(DecoratedOperator, DockerOperator):
     def __init__(
         self,
         use_dill=False,
+        python_command='python3',
         **kwargs,
     ) -> None:
         command = "dummy command"
+        self.python_command = python_command
         self.pickling_library = dill if use_dill else pickle
         super().__init__(
             command=command, retrieve_output=True, retrieve_output_path="/tmp/script.out", **kwargs
         )
 
+    def generate_command(self):
+        return (
+            f"""bash -cx  '{_generate_decode_command("__PYTHON_SCRIPT", "/tmp/script.py",
+                                                     self.python_command)} &&"""
+            f'{_generate_decode_command("__PYTHON_INPUT", "/tmp/script.in", self.python_command)} &&'
+            f'{self.python_command} /tmp/script.py /tmp/script.in /tmp/script.out\''
+        )
+
     def execute(self, context: 'Context'):
         with TemporaryDirectory(prefix='venv') as tmp_dir:
             input_filename = os.path.join(tmp_dir, 'script.in')
@@ -109,11 +119,7 @@ class _DockerDecoratedOperator(DecoratedOperator, DockerOperator):
             else:
                 self.environment["__PYTHON_INPUT"] = ""
 
-            self.command = (
-                f"""bash -cx  '{_generate_decode_command("__PYTHON_SCRIPT", "/tmp/script.py")} &&"""
-                f'{_generate_decode_command("__PYTHON_INPUT", "/tmp/script.in")} &&'
-                f'python /tmp/script.py /tmp/script.in /tmp/script.out\''
-            )
+            self.command = self.generate_command()
             return super().execute(context)
 
     def _get_python_source(self):
diff --git a/airflow/providers/docker/example_dags/example_docker.py b/airflow/providers/docker/example_dags/example_docker.py
index fba2c43..83f6744 100644
--- a/airflow/providers/docker/example_dags/example_docker.py
+++ b/airflow/providers/docker/example_dags/example_docker.py
@@ -34,7 +34,6 @@ t1 = BashOperator(task_id='print_date', bash_command='date', dag=dag)
 t2 = BashOperator(task_id='sleep', bash_command='sleep 5', retries=3, dag=dag)
 
 t3 = DockerOperator(
-    api_version='1.19',
     docker_url='tcp://localhost:2375',  # Set your docker URL
     command='/bin/sleep 30',
     image='centos:latest',