rkhachatryan commented on code in PR #21981:
URL: https://github.com/apache/flink/pull/21981#discussion_r1116813631


##########
flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/adaptive/JobSchedulingPlan.java:
##########
@@ -0,0 +1,97 @@
+/*
+ * 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.runtime.scheduler.adaptive;
+
+import org.apache.flink.annotation.Internal;
+import org.apache.flink.runtime.jobmaster.SlotInfo;
+import org.apache.flink.runtime.scheduler.adaptive.allocator.VertexParallelism;
+
+import java.util.Collection;
+
+/**
+ * A plan that describes how to execute {@link 
org.apache.flink.runtime.jobgraph.JobGraph JobGraph}.
+ *
+ * <ol>
+ *   <li>{@link #vertexParallelism} is necessary to create {@link
+ *       org.apache.flink.runtime.executiongraph.ExecutionGraph ExecutionGraph}
+ *   <li>{@link #slotAssignments} are used to schedule it onto the cluster
+ * </ol>
+ *
+ * {@link AdaptiveScheduler} passes this structure from {@link 
WaitingForResources} to {@link
+ * CreatingExecutionGraph} stages.
+ */
+@Internal
+public class JobSchedulingPlan {
+    private final VertexParallelism vertexParallelism;
+    private final Collection<SlotAssignment> slotAssignments;
+
+    public JobSchedulingPlan(
+            VertexParallelism vertexParallelism, Collection<SlotAssignment> 
slotAssignments) {
+        this.vertexParallelism = vertexParallelism;
+        this.slotAssignments = slotAssignments;
+    }
+
+    public VertexParallelism getVertexParallelism() {
+        return vertexParallelism;
+    }
+
+    public Collection<SlotAssignment> getSlotAssignments() {
+        return slotAssignments;
+    }
+
+    /** Assignment of a slot to some target (e.g. a slot sharing group). */
+    public static class SlotAssignment {
+        private final SlotInfo slotInfo;
+        /**
+         * Interpreted by {@link
+         * 
org.apache.flink.runtime.scheduler.adaptive.allocator.SlotAllocator#tryReserveResources(JobSchedulingPlan)}.
+         * This can be a slot sharing group, a task, or something else.
+         */
+        private final Object target;

Review Comment:
   Yes, but I wanted to keep the ability to change that. Currently (in 
`master`), other types are allowed by implementing `VertexParallelism`.



-- 
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: issues-unsubscr...@flink.apache.org

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

Reply via email to