You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@flink.apache.org by GitBox <gi...@apache.org> on 2022/03/04 15:29:29 UTC

[GitHub] [flink-kubernetes-operator] gyfora commented on a change in pull request #41: [FLINK-26377] Extract the Reconciler interface

gyfora commented on a change in pull request #41:
URL: https://github.com/apache/flink-kubernetes-operator/pull/41#discussion_r819666628



##########
File path: flink-kubernetes-operator/src/main/java/org/apache/flink/kubernetes/operator/reconciler/FlinkReconciler.java
##########
@@ -0,0 +1,45 @@
+/*
+ * 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.kubernetes.operator.reconciler;
+
+import org.apache.flink.configuration.Configuration;
+import org.apache.flink.kubernetes.operator.crd.FlinkDeployment;
+
+import io.javaoperatorsdk.operator.api.reconciler.Context;
+import io.javaoperatorsdk.operator.api.reconciler.DeleteControl;
+import io.javaoperatorsdk.operator.api.reconciler.UpdateControl;
+
+/**
+ * The internal interface of reconciler, It aligns to the {@link
+ * io.javaoperatorsdk.operator.api.reconciler.Reconciler}.
+ */
+public interface FlinkReconciler {
+
+    UpdateControl<FlinkDeployment> reconcile(
+            String operatorNamespace,
+            FlinkDeployment flinkApp,
+            Context context,
+            Configuration effectiveConfig)
+            throws Exception;
+
+    default DeleteControl cleanup(

Review comment:
       Why does this have a default basically empty implementaiton? I think we should keep it abstract (required)

##########
File path: flink-kubernetes-operator/src/main/java/org/apache/flink/kubernetes/operator/reconciler/CompositeReconciler.java
##########
@@ -0,0 +1,86 @@
+/*
+ * 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.kubernetes.operator.reconciler;
+
+import org.apache.flink.configuration.Configuration;
+import org.apache.flink.kubernetes.operator.config.FlinkOperatorConfiguration;
+import org.apache.flink.kubernetes.operator.crd.FlinkDeployment;
+import org.apache.flink.kubernetes.operator.service.FlinkService;
+
+import io.fabric8.kubernetes.client.KubernetesClient;
+import io.javaoperatorsdk.operator.api.reconciler.Context;
+import io.javaoperatorsdk.operator.api.reconciler.DeleteControl;
+import io.javaoperatorsdk.operator.api.reconciler.UpdateControl;
+
+import java.util.Map;
+import java.util.concurrent.ConcurrentHashMap;
+
+/** The composite reconciler which will use the target reconciler based on the app mode. */
+public class CompositeReconciler implements FlinkReconciler {
+
+    private final Map<Mode, FlinkReconciler> reconcilerMap = new ConcurrentHashMap<>();
+    private final KubernetesClient kubernetesClient;
+    private final FlinkService flinkService;
+    private final FlinkOperatorConfiguration operatorConfiguration;
+    private final FlinkReconcilerFactory factory;
+
+    public CompositeReconciler(
+            KubernetesClient kubernetesClient,
+            FlinkService flinkService,
+            FlinkOperatorConfiguration operatorConfiguration,
+            FlinkReconcilerFactory factory) {
+        this.kubernetesClient = kubernetesClient;
+        this.flinkService = flinkService;
+        this.operatorConfiguration = operatorConfiguration;
+        this.factory = factory;
+    }
+
+    @Override
+    public UpdateControl<FlinkDeployment> reconcile(
+            String operatorNamespace,
+            FlinkDeployment flinkApp,
+            Context context,
+            Configuration effectiveConfig)
+            throws Exception {
+        Mode mode = inferMode(flinkApp);
+        return getOrCreateReconciler(mode)
+                .reconcile(operatorNamespace, flinkApp, context, effectiveConfig);
+    }
+
+    @Override
+    public DeleteControl cleanup(
+            String operatorNamespace, FlinkDeployment flinkApp, Configuration effectiveConfig) {
+        Mode mode = inferMode(flinkApp);
+        return getOrCreateReconciler(mode).cleanup(operatorNamespace, flinkApp, effectiveConfig);
+    }
+
+    private FlinkReconciler getOrCreateReconciler(Mode mode) {
+        return reconcilerMap.computeIfAbsent(
+                mode,
+                m -> factory.create(kubernetesClient, flinkService, operatorConfiguration, mode));
+    }
+
+    private Mode inferMode(FlinkDeployment flinkApp) {

Review comment:
       I don't really see the value of having a Mode enum, it just adds unnecessary complexity, this simple if branch could be part of the factory logic, then we wouldnt be restricted to hardcoded set of modes

##########
File path: flink-kubernetes-operator/src/main/java/org/apache/flink/kubernetes/operator/reconciler/CompositeReconciler.java
##########
@@ -0,0 +1,86 @@
+/*
+ * 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.kubernetes.operator.reconciler;
+
+import org.apache.flink.configuration.Configuration;
+import org.apache.flink.kubernetes.operator.config.FlinkOperatorConfiguration;
+import org.apache.flink.kubernetes.operator.crd.FlinkDeployment;
+import org.apache.flink.kubernetes.operator.service.FlinkService;
+
+import io.fabric8.kubernetes.client.KubernetesClient;
+import io.javaoperatorsdk.operator.api.reconciler.Context;
+import io.javaoperatorsdk.operator.api.reconciler.DeleteControl;
+import io.javaoperatorsdk.operator.api.reconciler.UpdateControl;
+
+import java.util.Map;
+import java.util.concurrent.ConcurrentHashMap;
+
+/** The composite reconciler which will use the target reconciler based on the app mode. */
+public class CompositeReconciler implements FlinkReconciler {

Review comment:
       I think if we get rid of the mode we dont even need the CompositeReconciler and use the ReconcilerFactory in place of 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.

To unsubscribe, e-mail: commits-unsubscribe@flink.apache.org

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