You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@airflow.apache.org by GitBox <gi...@apache.org> on 2020/08/31 03:56:07 UTC

[GitHub] [airflow] pingzh opened a new pull request #10654: Add CeleryKubernetesExecutor

pingzh opened a new pull request #10654:
URL: https://github.com/apache/airflow/pull/10654


   it consists of CeleryExecutor and KubernetesExecutor, which allows users
   to route their tasks to either Kubernetes or Celery based on the queue
   defined on a task
   
   <!--
   Thank you for contributing! Please make sure that your code changes
   are covered with tests. And in case of new features or big changes
   remember to adjust the documentation.
   
   Feel free to ping committers for the review!
   
   In case of existing issue, reference it using one of the following:
   
   closes: #ISSUE
   related: #ISSUE
   
   How to write a good git commit message:
   http://chris.beams.io/posts/git-commit/
   -->
   
   ---
   **^ Add meaningful description above**
   
   Read the **[Pull Request Guidelines](https://github.com/apache/airflow/blob/master/CONTRIBUTING.rst#pull-request-guidelines)** for more information.
   In case of fundamental code change, Airflow Improvement Proposal ([AIP](https://cwiki.apache.org/confluence/display/AIRFLOW/Airflow+Improvements+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 [UPDATING.md](https://github.com/apache/airflow/blob/master/UPDATING.md).
   


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

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



[GitHub] [airflow] pingzh commented on pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
pingzh commented on pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#issuecomment-683575898


   Hi @kaxil and @dimberman 
   
   could you please take a look at this PR? I think I also need another pr to indicate which methods are private in the `executor` so that those methods are not supposed to be invoked outside the executor.  Please let me know your thoughts on this.


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

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



[GitHub] [airflow] potiuk commented on pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
potiuk commented on pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#issuecomment-691514332


   > Strange, i started to get this static check error after I rebased the master:
   
   Please rebase it again. We had a MyPy error kicking in and it was fixed in #10879 


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

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



[GitHub] [airflow] potiuk edited a comment on pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
potiuk edited a comment on pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#issuecomment-689472731


   > @potiuk Worth resurrecting your "multi-executor" PR from earlier this year to make it more generic, or do you think this is Good Enough?
   
   I think Multi-executor was too-generic. There are hardly use cases for a lot of combinations of the executors and the increased complexity of the configuration and potential side effects and issues if someone tries to combine for example Sequential Executor with KubernetesExecutor are well.... scary. 
   
   Also, it gives a very good message IMHO -> if you are in Production use Celery and Kubernetes executor. Full Stop.


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

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



[GitHub] [airflow] pingzh commented on pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
pingzh commented on pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#issuecomment-691523290


   (I re-forked the apache/airflow repo, mine was the incubator-airflow)
   
   the new  PR is: #10901 


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

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



[GitHub] [airflow] turbaszek commented on a change in pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
turbaszek commented on a change in pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#discussion_r486617206



##########
File path: airflow/executors/celery_kubernetes_executor.py
##########
@@ -0,0 +1,157 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Dict, Optional, Set, Union
+
+from airflow.configuration import conf
+from airflow.executors.base_executor import CommandType, EventBufferValueType, QueuedTaskInstanceType
+from airflow.executors.celery_executor import CeleryExecutor
+from airflow.executors.kubernetes_executor import KubernetesExecutor
+from airflow.models.taskinstance import SimpleTaskInstance, TaskInstance, TaskInstanceKey
+from airflow.utils.log.logging_mixin import LoggingMixin
+
+
+class CeleryKubernetesExecutor(LoggingMixin):

Review comment:
       Ok I have to do this 😄 I was thinking about this yesterday but was not sure... Do you think that this name is good? On one hand, it's explicit on the other... I can imagine questions like "Is this for Celery on Kubernetes?", "Can I use CeleryExecutor on kubernetes? Or do I have to use this one?"
   
   Do I have any better ideas? `CeleryAndKubernetesExecutor`? Is this better? I don't know just want to hear what others think 🤔 




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

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



[GitHub] [airflow] dimberman commented on a change in pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
dimberman commented on a change in pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#discussion_r486500203



##########
File path: airflow/executors/celery_kubernetes_executor.py
##########
@@ -0,0 +1,156 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Dict, Optional, Set, Union
+
+from airflow.executors.base_executor import CommandType, EventBufferValueType, QueuedTaskInstanceType
+from airflow.executors.celery_executor import CeleryExecutor
+from airflow.executors.kubernetes_executor import KubernetesExecutor
+from airflow.models.taskinstance import SimpleTaskInstance, TaskInstance, TaskInstanceKey
+from airflow.utils.log.logging_mixin import LoggingMixin
+
+
+class CeleryKubernetesExecutor(LoggingMixin):
+    """
+    CeleryKubernetesExecutor consists of celery_executor and kubernetes_executor.
+    It chooses an executor to use based on the queue defined on the task.
+    When the queue is `kubernetes`, kubernetes_executor is selected to run the task,
+    otherwise, celery_executor is used.
+    """
+
+    KUBERNETES_QUEUE = 'kubernetes'

Review comment:
       I agree the default should be "kubernetes" though.




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

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



[GitHub] [airflow] pingzh commented on pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
pingzh commented on pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#issuecomment-690772877


   Strange, i started to get this static check error after I rebased the master:
   
   ```
   Run mypy..........................................................................................................................Failed
   - hook id: mypy
   - exit code: 1
   
   airflow/models/dagrun.py:72: error: unused 'type: ignore' comment
               primaryjoin=and_(TI.dag_id == dag_id, TI.execution_date == exe...
               ^
   Found 1 error in 1 file (checked 2408 source files)
   ```


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

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



[GitHub] [airflow] ashb commented on pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
ashb commented on pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#issuecomment-689463060


   @potiuk Worth resurrecting your "multi-executor" PR from earlier this year to make it more generic, or do you think this is Good Enough?


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

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



[GitHub] [airflow] pingzh commented on a change in pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
pingzh commented on a change in pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#discussion_r486651631



##########
File path: airflow/executors/celery_kubernetes_executor.py
##########
@@ -0,0 +1,157 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Dict, Optional, Set, Union
+
+from airflow.configuration import conf
+from airflow.executors.base_executor import CommandType, EventBufferValueType, QueuedTaskInstanceType
+from airflow.executors.celery_executor import CeleryExecutor
+from airflow.executors.kubernetes_executor import KubernetesExecutor
+from airflow.models.taskinstance import SimpleTaskInstance, TaskInstance, TaskInstanceKey
+from airflow.utils.log.logging_mixin import LoggingMixin
+
+
+class CeleryKubernetesExecutor(LoggingMixin):

Review comment:
       Good point about the naming.  i like it, `CeleryAndKubernetesExecutor` which indicates `celery` and `kubernetes` are both used in this class.




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

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



[GitHub] [airflow] potiuk commented on pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
potiuk commented on pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#issuecomment-691514332


   > Strange, i started to get this static check error after I rebased the master:
   
   Please rebase it again. We had a MyPy error kicking in and it was fixed in #10879 


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

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



[GitHub] [airflow] kaxil commented on a change in pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
kaxil commented on a change in pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#discussion_r486653277



##########
File path: airflow/executors/celery_kubernetes_executor.py
##########
@@ -0,0 +1,157 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Dict, Optional, Set, Union
+
+from airflow.configuration import conf
+from airflow.executors.base_executor import CommandType, EventBufferValueType, QueuedTaskInstanceType
+from airflow.executors.celery_executor import CeleryExecutor
+from airflow.executors.kubernetes_executor import KubernetesExecutor
+from airflow.models.taskinstance import SimpleTaskInstance, TaskInstance, TaskInstanceKey
+from airflow.utils.log.logging_mixin import LoggingMixin
+
+
+class CeleryKubernetesExecutor(LoggingMixin):

Review comment:
       No strong opinion on the naming of the class, I am fine with either since we have the first line of the docstring explaining it :)




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

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



[GitHub] [airflow] potiuk commented on pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
potiuk commented on pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#issuecomment-691514332


   > Strange, i started to get this static check error after I rebased the master:
   
   Please rebase it again. We had a MyPy error kicking in and it was fixed in #10879 


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

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



[GitHub] [airflow] turbaszek commented on pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
turbaszek commented on pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#issuecomment-691537085






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

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



[GitHub] [airflow] dimberman commented on a change in pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
dimberman commented on a change in pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#discussion_r486677273



##########
File path: airflow/executors/celery_kubernetes_executor.py
##########
@@ -0,0 +1,157 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Dict, Optional, Set, Union
+
+from airflow.configuration import conf
+from airflow.executors.base_executor import CommandType, EventBufferValueType, QueuedTaskInstanceType
+from airflow.executors.celery_executor import CeleryExecutor
+from airflow.executors.kubernetes_executor import KubernetesExecutor
+from airflow.models.taskinstance import SimpleTaskInstance, TaskInstance, TaskInstanceKey
+from airflow.utils.log.logging_mixin import LoggingMixin
+
+
+class CeleryKubernetesExecutor(LoggingMixin):

Review comment:
       @turbaszek @pingzh Maybe we can name it the "MultiExecutor" or "JoinedExecutor"? I could see a case where in the future we route even more executors into this (like Dask)




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

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



[GitHub] [airflow] pingzh commented on pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
pingzh commented on pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#issuecomment-691523290






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

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



[GitHub] [airflow] turbaszek commented on pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
turbaszek commented on pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#issuecomment-691537085






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

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



[GitHub] [airflow] pingzh commented on a change in pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
pingzh commented on a change in pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#discussion_r486788268



##########
File path: airflow/executors/celery_kubernetes_executor.py
##########
@@ -0,0 +1,157 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Dict, Optional, Set, Union
+
+from airflow.configuration import conf
+from airflow.executors.base_executor import CommandType, EventBufferValueType, QueuedTaskInstanceType
+from airflow.executors.celery_executor import CeleryExecutor
+from airflow.executors.kubernetes_executor import KubernetesExecutor
+from airflow.models.taskinstance import SimpleTaskInstance, TaskInstance, TaskInstanceKey
+from airflow.utils.log.logging_mixin import LoggingMixin
+
+
+class CeleryKubernetesExecutor(LoggingMixin):

Review comment:
       @dimberman thanks for the suggestion. I think we can go further to have a `BaseComposeddExecutor`, which serves as the base class for any composed executor. The only two methods for the subclass to override is the `__init__` and also the `_route` method.
   
   However, I am thinking of whether we over-optimize it in this case. I don't want to have a too generic name at the beginning. We can come back to refactor the logic when we have another use case like this. 
   

##########
File path: airflow/executors/celery_kubernetes_executor.py
##########
@@ -0,0 +1,157 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Dict, Optional, Set, Union
+
+from airflow.configuration import conf
+from airflow.executors.base_executor import CommandType, EventBufferValueType, QueuedTaskInstanceType
+from airflow.executors.celery_executor import CeleryExecutor
+from airflow.executors.kubernetes_executor import KubernetesExecutor
+from airflow.models.taskinstance import SimpleTaskInstance, TaskInstance, TaskInstanceKey
+from airflow.utils.log.logging_mixin import LoggingMixin
+
+
+class CeleryKubernetesExecutor(LoggingMixin):

Review comment:
       @dimberman thanks for the suggestion. I think we can go further to have a `BaseComposeddExecutor`, which serves as the base class for any composed executor. The only two methods for the subclass to override is the `__init__` and also the `_route` method.
   
   However, I am thinking of whether we over-optimize it in this case. I don't want to have a too generic name at the beginning. We can come back to refactor the logic when we have another use case like this. 
   

##########
File path: airflow/executors/celery_kubernetes_executor.py
##########
@@ -0,0 +1,157 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Dict, Optional, Set, Union
+
+from airflow.configuration import conf
+from airflow.executors.base_executor import CommandType, EventBufferValueType, QueuedTaskInstanceType
+from airflow.executors.celery_executor import CeleryExecutor
+from airflow.executors.kubernetes_executor import KubernetesExecutor
+from airflow.models.taskinstance import SimpleTaskInstance, TaskInstance, TaskInstanceKey
+from airflow.utils.log.logging_mixin import LoggingMixin
+
+
+class CeleryKubernetesExecutor(LoggingMixin):

Review comment:
       @dimberman thanks for the suggestion. I think we can go further to have a `BaseComposeddExecutor`, which serves as the base class for any composed executor. The only two methods for the subclass to override is the `__init__` and also the `_route` method.
   
   However, I am thinking of whether we over-optimize it in this case. I don't want to have a too generic name at the beginning. We can come back to refactor the logic when we have another use case like this. 
   

##########
File path: airflow/executors/celery_kubernetes_executor.py
##########
@@ -0,0 +1,157 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Dict, Optional, Set, Union
+
+from airflow.configuration import conf
+from airflow.executors.base_executor import CommandType, EventBufferValueType, QueuedTaskInstanceType
+from airflow.executors.celery_executor import CeleryExecutor
+from airflow.executors.kubernetes_executor import KubernetesExecutor
+from airflow.models.taskinstance import SimpleTaskInstance, TaskInstance, TaskInstanceKey
+from airflow.utils.log.logging_mixin import LoggingMixin
+
+
+class CeleryKubernetesExecutor(LoggingMixin):

Review comment:
       @dimberman thanks for the suggestion. I think we can go further to have a `BaseComposeddExecutor`, which serves as the base class for any composed executor. The only two methods for the subclass to override is the `__init__` and also the `_route` method.
   
   However, I am thinking of whether we over-optimize it in this case. I don't want to have a too generic name at the beginning. We can come back to refactor the logic when we have another use case like this. 
   




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

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



[GitHub] [airflow] turbaszek commented on pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
turbaszek commented on pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#issuecomment-691537085


   @pingzh should we close this one?


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

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



[GitHub] [airflow] pingzh closed pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
pingzh closed pull request #10654:
URL: https://github.com/apache/airflow/pull/10654


   


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

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



[GitHub] [airflow] potiuk commented on pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
potiuk commented on pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#issuecomment-689472731


   > @potiuk Worth resurrecting your "multi-executor" PR from earlier this year to make it more generic, or do you think this is Good Enough?
   
   I think Multi-executor was too-generic. There are hardly use cases for a lot of combinations of the executors and the increased complexity of the configuration and potential side effects and issue if someone tries to combine for example Sequential Executor with KubernetesExecutor are well.... scary. 
   
   Also, it gives a very good message IMHO -> if you are in Production use Celery and Kubernetes executor. Full Stop.


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

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



[GitHub] [airflow] turbaszek commented on a change in pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
turbaszek commented on a change in pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#discussion_r485585205



##########
File path: airflow/executors/celery_kubernetes_executor.py
##########
@@ -0,0 +1,156 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Dict, Optional, Set, Union
+
+from airflow.executors.base_executor import CommandType, EventBufferValueType, QueuedTaskInstanceType
+from airflow.executors.celery_executor import CeleryExecutor
+from airflow.executors.kubernetes_executor import KubernetesExecutor
+from airflow.models.taskinstance import SimpleTaskInstance, TaskInstance, TaskInstanceKey
+from airflow.utils.log.logging_mixin import LoggingMixin
+
+
+class CeleryKubernetesExecutor(LoggingMixin):
+    """
+    CeleryKubernetesExecutor consists of celery_executor and kubernetes_executor.

Review comment:
       ```suggestion
       CeleryKubernetesExecutor consists of CeleryExecutor and KubernetesExecutor.
   ```




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

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



[GitHub] [airflow] pingzh commented on a change in pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
pingzh commented on a change in pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#discussion_r486788268



##########
File path: airflow/executors/celery_kubernetes_executor.py
##########
@@ -0,0 +1,157 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Dict, Optional, Set, Union
+
+from airflow.configuration import conf
+from airflow.executors.base_executor import CommandType, EventBufferValueType, QueuedTaskInstanceType
+from airflow.executors.celery_executor import CeleryExecutor
+from airflow.executors.kubernetes_executor import KubernetesExecutor
+from airflow.models.taskinstance import SimpleTaskInstance, TaskInstance, TaskInstanceKey
+from airflow.utils.log.logging_mixin import LoggingMixin
+
+
+class CeleryKubernetesExecutor(LoggingMixin):

Review comment:
       @dimberman thanks for the suggestion. I think we can go further to have a `BaseComposeddExecutor`, which serves as the base class for any composed executor. The only two methods for the subclass to override is the `__init__` and also the `_route` method.
   
   However, I am thinking of whether we over-optimize it in this case. I don't want to have a too generic name at the beginning. We can come back to refactor the logic when we have another use case like this. 
   




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

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



[GitHub] [airflow] pingzh commented on a change in pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
pingzh commented on a change in pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#discussion_r486788268



##########
File path: airflow/executors/celery_kubernetes_executor.py
##########
@@ -0,0 +1,157 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Dict, Optional, Set, Union
+
+from airflow.configuration import conf
+from airflow.executors.base_executor import CommandType, EventBufferValueType, QueuedTaskInstanceType
+from airflow.executors.celery_executor import CeleryExecutor
+from airflow.executors.kubernetes_executor import KubernetesExecutor
+from airflow.models.taskinstance import SimpleTaskInstance, TaskInstance, TaskInstanceKey
+from airflow.utils.log.logging_mixin import LoggingMixin
+
+
+class CeleryKubernetesExecutor(LoggingMixin):

Review comment:
       @dimberman thanks for the suggestion. I think we can go further to have a `BaseComposeddExecutor`, which serves as the base class for any composed executor. The only two methods for the subclass to override is the `__init__` and also the `_route` method.
   
   However, I am thinking of whether we over-optimize it in this case. I don't want to have a too generic name at the beginning. We can come back to refactor the logic when we have another use case like this. 
   




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

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



[GitHub] [airflow] pingzh closed pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
pingzh closed pull request #10654:
URL: https://github.com/apache/airflow/pull/10654






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

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



[GitHub] [airflow] pingzh commented on a change in pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
pingzh commented on a change in pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#discussion_r486788268



##########
File path: airflow/executors/celery_kubernetes_executor.py
##########
@@ -0,0 +1,157 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Dict, Optional, Set, Union
+
+from airflow.configuration import conf
+from airflow.executors.base_executor import CommandType, EventBufferValueType, QueuedTaskInstanceType
+from airflow.executors.celery_executor import CeleryExecutor
+from airflow.executors.kubernetes_executor import KubernetesExecutor
+from airflow.models.taskinstance import SimpleTaskInstance, TaskInstance, TaskInstanceKey
+from airflow.utils.log.logging_mixin import LoggingMixin
+
+
+class CeleryKubernetesExecutor(LoggingMixin):

Review comment:
       @dimberman thanks for the suggestion. I think we can go further to have a `BaseComposeddExecutor`, which serves as the base class for any composed executor. The only two methods for the subclass to override is the `__init__` and also the `_route` method.
   
   However, I am thinking of whether we over-optimize it in this case. I don't want to have a too generic name at the beginning. We can come back to refactor the logic when we have another use case like this. 
   

##########
File path: airflow/executors/celery_kubernetes_executor.py
##########
@@ -0,0 +1,157 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Dict, Optional, Set, Union
+
+from airflow.configuration import conf
+from airflow.executors.base_executor import CommandType, EventBufferValueType, QueuedTaskInstanceType
+from airflow.executors.celery_executor import CeleryExecutor
+from airflow.executors.kubernetes_executor import KubernetesExecutor
+from airflow.models.taskinstance import SimpleTaskInstance, TaskInstance, TaskInstanceKey
+from airflow.utils.log.logging_mixin import LoggingMixin
+
+
+class CeleryKubernetesExecutor(LoggingMixin):

Review comment:
       @dimberman thanks for the suggestion. I think we can go further to have a `BaseComposeddExecutor`, which serves as the base class for any composed executor. The only two methods for the subclass to override is the `__init__` and also the `_route` method.
   
   However, I am thinking of whether we over-optimize it in this case. I don't want to have a too generic name at the beginning. We can come back to refactor the logic when we have another use case like this. 
   

##########
File path: airflow/executors/celery_kubernetes_executor.py
##########
@@ -0,0 +1,157 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Dict, Optional, Set, Union
+
+from airflow.configuration import conf
+from airflow.executors.base_executor import CommandType, EventBufferValueType, QueuedTaskInstanceType
+from airflow.executors.celery_executor import CeleryExecutor
+from airflow.executors.kubernetes_executor import KubernetesExecutor
+from airflow.models.taskinstance import SimpleTaskInstance, TaskInstance, TaskInstanceKey
+from airflow.utils.log.logging_mixin import LoggingMixin
+
+
+class CeleryKubernetesExecutor(LoggingMixin):

Review comment:
       @dimberman thanks for the suggestion. I think we can go further to have a `BaseComposeddExecutor`, which serves as the base class for any composed executor. The only two methods for the subclass to override is the `__init__` and also the `_route` method.
   
   However, I am thinking of whether we over-optimize it in this case. I don't want to have a too generic name at the beginning. We can come back to refactor the logic when we have another use case like this. 
   

##########
File path: airflow/executors/celery_kubernetes_executor.py
##########
@@ -0,0 +1,157 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Dict, Optional, Set, Union
+
+from airflow.configuration import conf
+from airflow.executors.base_executor import CommandType, EventBufferValueType, QueuedTaskInstanceType
+from airflow.executors.celery_executor import CeleryExecutor
+from airflow.executors.kubernetes_executor import KubernetesExecutor
+from airflow.models.taskinstance import SimpleTaskInstance, TaskInstance, TaskInstanceKey
+from airflow.utils.log.logging_mixin import LoggingMixin
+
+
+class CeleryKubernetesExecutor(LoggingMixin):

Review comment:
       @dimberman thanks for the suggestion. I think we can go further to have a `BaseComposeddExecutor`, which serves as the base class for any composed executor. The only two methods for the subclass to override is the `__init__` and also the `_route` method.
   
   However, I am thinking of whether we over-optimize it in this case. I don't want to have a too generic name at the beginning. We can come back to refactor the logic when we have another use case like this. 
   

##########
File path: airflow/executors/celery_kubernetes_executor.py
##########
@@ -0,0 +1,157 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Dict, Optional, Set, Union
+
+from airflow.configuration import conf
+from airflow.executors.base_executor import CommandType, EventBufferValueType, QueuedTaskInstanceType
+from airflow.executors.celery_executor import CeleryExecutor
+from airflow.executors.kubernetes_executor import KubernetesExecutor
+from airflow.models.taskinstance import SimpleTaskInstance, TaskInstance, TaskInstanceKey
+from airflow.utils.log.logging_mixin import LoggingMixin
+
+
+class CeleryKubernetesExecutor(LoggingMixin):

Review comment:
       @dimberman thanks for the suggestion. I think we can go further to have a `BaseComposeddExecutor`, which serves as the base class for any composed executor. The only two methods for the subclass to override is the `__init__` and also the `_route` method.
   
   However, I am thinking of whether we over-optimize it in this case. I don't want to have a too generic name at the beginning. We can come back to refactor the logic when we have another use case like this. 
   




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

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



[GitHub] [airflow] turbaszek commented on a change in pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
turbaszek commented on a change in pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#discussion_r485580885



##########
File path: airflow/executors/celery_kubernetes_executor.py
##########
@@ -0,0 +1,156 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Dict, Optional, Set, Union
+
+from airflow.executors.base_executor import CommandType, EventBufferValueType, QueuedTaskInstanceType
+from airflow.executors.celery_executor import CeleryExecutor
+from airflow.executors.kubernetes_executor import KubernetesExecutor
+from airflow.models.taskinstance import SimpleTaskInstance, TaskInstance, TaskInstanceKey
+from airflow.utils.log.logging_mixin import LoggingMixin
+
+
+class CeleryKubernetesExecutor(LoggingMixin):
+    """
+    CeleryKubernetesExecutor consists of celery_executor and kubernetes_executor.
+    It chooses an executor to use based on the queue defined on the task.
+    When the queue is `kubernetes`, kubernetes_executor is selected to run the task,
+    otherwise, celery_executor is used.
+    """
+
+    KUBERNETES_QUEUE = 'kubernetes'

Review comment:
       Should this be configurable? What do you think?




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

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



[GitHub] [airflow] pingzh commented on a change in pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
pingzh commented on a change in pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#discussion_r486486640



##########
File path: airflow/executors/celery_kubernetes_executor.py
##########
@@ -0,0 +1,156 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Dict, Optional, Set, Union
+
+from airflow.executors.base_executor import CommandType, EventBufferValueType, QueuedTaskInstanceType
+from airflow.executors.celery_executor import CeleryExecutor
+from airflow.executors.kubernetes_executor import KubernetesExecutor
+from airflow.models.taskinstance import SimpleTaskInstance, TaskInstance, TaskInstanceKey
+from airflow.utils.log.logging_mixin import LoggingMixin
+
+
+class CeleryKubernetesExecutor(LoggingMixin):
+    """
+    CeleryKubernetesExecutor consists of celery_executor and kubernetes_executor.
+    It chooses an executor to use based on the queue defined on the task.
+    When the queue is `kubernetes`, kubernetes_executor is selected to run the task,
+    otherwise, celery_executor is used.
+    """
+
+    KUBERNETES_QUEUE = 'kubernetes'

Review comment:
       Agree 👍 




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

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



[GitHub] [airflow] pingzh commented on pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
pingzh commented on pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#issuecomment-691523290






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

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



[GitHub] [airflow] turbaszek commented on pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
turbaszek commented on pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#issuecomment-689539291


   +1 for this approach instead of "generic-multi-executor". @pingzh can you add please some documentation under `docs/executor`?


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

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



[GitHub] [airflow] turbaszek commented on a change in pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
turbaszek commented on a change in pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#discussion_r485585583



##########
File path: airflow/executors/celery_kubernetes_executor.py
##########
@@ -0,0 +1,156 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one
+# or more contributor license agreements.  See the NOTICE file
+# distributed with this work for additional information
+# regarding copyright ownership.  The ASF licenses this file
+# to you under the Apache License, Version 2.0 (the
+# "License"); you may not use this file except in compliance
+# with the License.  You may obtain a copy of the License at
+#
+#   http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing,
+# software distributed under the License is distributed on an
+# "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+# KIND, either express or implied.  See the License for the
+# specific language governing permissions and limitations
+# under the License.
+from typing import Dict, Optional, Set, Union
+
+from airflow.executors.base_executor import CommandType, EventBufferValueType, QueuedTaskInstanceType
+from airflow.executors.celery_executor import CeleryExecutor
+from airflow.executors.kubernetes_executor import KubernetesExecutor
+from airflow.models.taskinstance import SimpleTaskInstance, TaskInstance, TaskInstanceKey
+from airflow.utils.log.logging_mixin import LoggingMixin
+
+
+class CeleryKubernetesExecutor(LoggingMixin):
+    """
+    CeleryKubernetesExecutor consists of celery_executor and kubernetes_executor.
+    It chooses an executor to use based on the queue defined on the task.
+    When the queue is `kubernetes`, kubernetes_executor is selected to run the task,
+    otherwise, celery_executor is used.

Review comment:
       ```suggestion
       When the queue is `kubernetes`, KubernetesExecutor is selected to run the task,
       otherwise, CeleryExecutor is used.
   ```




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

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



[GitHub] [airflow] turbaszek commented on pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
turbaszek commented on pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#issuecomment-691537085


   @pingzh should we close this one?


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

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



[GitHub] [airflow] pingzh closed pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
pingzh closed pull request #10654:
URL: https://github.com/apache/airflow/pull/10654






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

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



[GitHub] [airflow] pingzh commented on pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
pingzh commented on pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#issuecomment-691537382


   > @pingzh should we close this one?
   
   Let us close this one. Sorry about this as it caused the comments not showing up in the new PR.


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

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



[GitHub] [airflow] potiuk commented on pull request #10654: Add CeleryKubernetesExecutor

Posted by GitBox <gi...@apache.org>.
potiuk commented on pull request #10654:
URL: https://github.com/apache/airflow/pull/10654#issuecomment-691514332






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

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