wanglijie95 commented on a change in pull request #18023:
URL: https://github.com/apache/flink/pull/18023#discussion_r773634628
##########
File path:
flink-runtime/src/test/java/org/apache/flink/runtime/executiongraph/ExecutionJobVertexTest.java
##########
@@ -93,6 +92,63 @@ public void testLazyInitialization() throws Exception {
assertThat(ejv.getTaskVertices().length, is(3));
assertThat(ejv.getInputs().size(), is(0));
assertThat(ejv.getProducedDataSets().length, is(1));
+ assertThat(ejv.getOperatorCoordinators().size(), is(0));
+ }
+
+ @Test(expected = IllegalStateException.class)
+ public void testErrorIfInitializationWithoutParallelismDecided() throws
Exception {
+ final ExecutionJobVertex ejv = createDynamicExecutionJobVertex();
+
+ initializeVertex(ejv);
+ }
+
+ @Test
+ public void testSetParallelismLazily() throws Exception {
+ final int parallelism = 3;
+ final int defaultMaxParallelism = 13;
+ final ExecutionJobVertex ejv =
+ createDynamicExecutionJobVertex(-1, -1, defaultMaxParallelism);
+
+ assertThat(ejv.isParallelismDecided(), is(false));
+
+ ejv.setParallelism(parallelism);
+
+ assertThat(ejv.isParallelismDecided(), is(true));
+ assertThat(ejv.getParallelism(), is(parallelism));
+
+ initializeVertex(ejv);
+
+ assertThat(ejv.getTaskVertices().length, is(parallelism));
+ }
+
+ @Test
+ public void testConfiguredMaxParallelismIsRespected() throws Exception {
+ final int configuredMaxParallelism = 12;
+ final int defaultMaxParallelism = 13;
+ final ExecutionJobVertex ejv =
+ createDynamicExecutionJobVertex(
+ -1, configuredMaxParallelism, defaultMaxParallelism);
+
+ assertThat(ejv.getMaxParallelism(), is(configuredMaxParallelism));
+ }
+
+ @Test
+ public void testComputingMaxParallelismFromConfiguredParallelism() throws
Exception {
+ final int parallelism = 300;
+ final int defaultMaxParallelism = 13;
+ final ExecutionJobVertex ejv =
+ createDynamicExecutionJobVertex(parallelism, -1,
defaultMaxParallelism);
+
+ assertThat(ejv.getMaxParallelism(), is(512));
+ }
+
+ @Test
+ public void testFallingBackToDefaultParallelism() throws Exception {
Review comment:
fixed
##########
File path:
flink-runtime/src/main/java/org/apache/flink/runtime/scheduler/SchedulerBase.java
##########
@@ -262,22 +264,55 @@ public static int getDefaultMaxParallelism(JobVertex
vertex) {
normalizeParallelism(vertex.getParallelism()));
}
+ // TODO: move to Adaptive Batch Scheduler.
+ /**
+ * Compute the {@link VertexParallelismStore} for all given vertices in a
dynamic graph, which
+ * will set defaults and ensure that the returned store contains valid
parallelisms, with the
+ * configured default max parallelism.
+ *
+ * @param vertices the vertices to compute parallelism for
+ * @param defaultMaxParallelism a function for computing a default max
parallelism if none is
Review comment:
fixed
##########
File path:
flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/ExecutionJobVertex.java
##########
@@ -307,13 +323,21 @@ public JobVertexID getJobVertexId() {
@Override
public ExecutionVertex[] getTaskVertices() {
+ if (taskVertices == null) {
+ LOG.warn(
+ "Trying to get execution vertices of an uninitialized job
vertex "
+ + getJobVertexId());
+ return new ExecutionVertex[0];
+ }
return taskVertices;
}
public IntermediateResult[] getProducedDataSets() {
+ checkState(isInitialized());
return producedDataSets;
}
+ @Nullable
public InputSplitAssigner getSplitAssigner() {
return splitAssigner;
Review comment:
fixed
--
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: [email protected]
For queries about this service, please contact Infrastructure at:
[email protected]