You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@flink.apache.org by kk...@apache.org on 2019/11/12 07:58:48 UTC

[flink] 08/24: [FLINK-XXXXX] Add the Executor-related interfaces.

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

kkloudas pushed a commit to branch executors
in repository https://gitbox.apache.org/repos/asf/flink.git

commit 5d734e6532076e870d22944bb75778b5d3547a40
Author: Kostas Kloudas <kk...@gmail.com>
AuthorDate: Fri Nov 1 11:31:24 2019 +0100

    [FLINK-XXXXX] Add the Executor-related interfaces.
---
 .../org/apache/flink/core/execution/Executor.java  | 37 ++++++++++++++++++++
 .../flink/core/execution/ExecutorFactory.java      | 40 ++++++++++++++++++++++
 .../core/execution/ExecutorServiceLoader.java      | 39 +++++++++++++++++++++
 3 files changed, 116 insertions(+)

diff --git a/flink-core/src/main/java/org/apache/flink/core/execution/Executor.java b/flink-core/src/main/java/org/apache/flink/core/execution/Executor.java
new file mode 100644
index 0000000..428ab00
--- /dev/null
+++ b/flink-core/src/main/java/org/apache/flink/core/execution/Executor.java
@@ -0,0 +1,37 @@
+/*
+ * 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.
+ */
+
+package org.apache.flink.core.execution;
+
+import org.apache.flink.api.common.JobExecutionResult;
+import org.apache.flink.api.dag.Pipeline;
+import org.apache.flink.configuration.Configuration;
+
+/**
+ * The entity responsible for executing a {@link Pipeline}, i.e. a user job.
+ */
+public interface Executor {
+
+	/**
+	 * Executes a {@link Pipeline} based on the provided configuration.
+	 * @param pipeline the {@link Pipeline} to execute
+	 * @param executionConfig the {@link Configuration} with the required execution parameters
+	 * @return the {@link JobExecutionResult} corresponding to the pipeline execution.
+	 */
+	JobExecutionResult execute(Pipeline pipeline, Configuration executionConfig) throws Exception;
+}
diff --git a/flink-core/src/main/java/org/apache/flink/core/execution/ExecutorFactory.java b/flink-core/src/main/java/org/apache/flink/core/execution/ExecutorFactory.java
new file mode 100644
index 0000000..8d6687b
--- /dev/null
+++ b/flink-core/src/main/java/org/apache/flink/core/execution/ExecutorFactory.java
@@ -0,0 +1,40 @@
+/*
+ * 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.
+ */
+
+package org.apache.flink.core.execution;
+
+import org.apache.flink.configuration.Configuration;
+
+/**
+ * A factory for selecting and instantiating the adequate {@link Executor}
+ * based on a provided {@link Configuration}.
+ */
+public interface ExecutorFactory {
+
+	/**
+	 * Returns {@code true} if this factory is compatible with the options in the
+	 * provided configuration, {@code false} otherwise.
+	 */
+	boolean isCompatibleWith(Configuration configuration);
+
+	/**
+	 * Instantiates an {@link Executor} compatible with the provided configuration.
+	 * @return the executor instance.
+	 */
+	Executor getExecutor(Configuration configuration);
+}
diff --git a/flink-core/src/main/java/org/apache/flink/core/execution/ExecutorServiceLoader.java b/flink-core/src/main/java/org/apache/flink/core/execution/ExecutorServiceLoader.java
new file mode 100644
index 0000000..0b959eb7
--- /dev/null
+++ b/flink-core/src/main/java/org/apache/flink/core/execution/ExecutorServiceLoader.java
@@ -0,0 +1,39 @@
+/*
+ * 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.
+ */
+
+package org.apache.flink.core.execution;
+
+import org.apache.flink.configuration.Configuration;
+
+/**
+ * An interface to be implemented by the entity responsible for finding the correct {@link Executor} to
+ * execute a given {@link org.apache.flink.api.dag.Pipeline}.
+ */
+public interface ExecutorServiceLoader {
+
+	/**
+	 * Loads the {@link ExecutorFactory} which is compatible with the provided configuration.
+	 * There can be at most one compatible factory among the available ones, otherwise an exception
+	 * will be thrown.
+	 *
+	 * @return a compatible {@link ExecutorFactory}.
+	 * @throws Exception if there is more than one compatible factories, or something went wrong when
+	 * 			loading the registered factories.
+	 */
+	ExecutorFactory getExecutorFactory(Configuration configuration) throws Exception;
+}