Skip to content

Commit 7ebc33c

Browse files
committed
Fix incorrect nullness in FlinkJobInvoker and JobInvoker
1 parent 5e5e147 commit 7ebc33c

File tree

2 files changed

+10
-12
lines changed

2 files changed

+10
-12
lines changed

runners/flink/src/main/java/org/apache/beam/runners/flink/FlinkJobInvoker.java

+8-9
Original file line numberDiff line numberDiff line change
@@ -17,6 +17,7 @@
1717
*/
1818
package org.apache.beam.runners.flink;
1919

20+
import static org.apache.beam.sdk.util.Preconditions.checkStateNotNull;
2021
import static org.apache.beam.sdk.util.construction.resources.PipelineResources.detectClassPathResourcesToStage;
2122

2223
import java.util.UUID;
@@ -31,14 +32,10 @@
3132
import org.apache.beam.vendor.grpc.v1p60p1.com.google.protobuf.Struct;
3233
import org.apache.beam.vendor.guava.v32_1_2_jre.com.google.common.base.Strings;
3334
import org.apache.beam.vendor.guava.v32_1_2_jre.com.google.common.util.concurrent.ListeningExecutorService;
34-
import org.checkerframework.checker.nullness.qual.Nullable;
3535
import org.slf4j.Logger;
3636
import org.slf4j.LoggerFactory;
3737

3838
/** Job Invoker for the {@link FlinkRunner}. */
39-
@SuppressWarnings({
40-
"nullness" // TODO(https://github.com/apache/beam/issues/20497)
41-
})
4239
public class FlinkJobInvoker extends JobInvoker {
4340
private static final Logger LOG = LoggerFactory.getLogger(FlinkJobInvoker.class);
4441

@@ -57,7 +54,7 @@ protected FlinkJobInvoker(FlinkJobServerDriver.FlinkServerConfiguration serverCo
5754
protected JobInvocation invokeWithExecutor(
5855
RunnerApi.Pipeline pipeline,
5956
Struct options,
60-
@Nullable String retrievalToken,
57+
String retrievalToken,
6158
ListeningExecutorService executorService) {
6259

6360
// TODO: How to make Java/Python agree on names of keys and their values?
@@ -74,20 +71,22 @@ protected JobInvocation invokeWithExecutor(
7471

7572
PortablePipelineOptions portableOptions = flinkOptions.as(PortablePipelineOptions.class);
7673

74+
ClassLoader thisClassLoader =
75+
checkStateNotNull(
76+
FlinkJobInvoker.class.getClassLoader(),
77+
"FlinkJobInvoker class loader is null - this means it was loaded by the bootstrap classloader, which should be impossible");
78+
7779
PortablePipelineRunner pipelineRunner;
7880
if (Strings.isNullOrEmpty(portableOptions.getOutputExecutablePath())) {
7981
pipelineRunner =
8082
new FlinkPipelineRunner(
8183
flinkOptions,
8284
serverConfig.getFlinkConfDir(),
83-
detectClassPathResourcesToStage(
84-
FlinkJobInvoker.class.getClassLoader(), flinkOptions));
85+
detectClassPathResourcesToStage(thisClassLoader, flinkOptions));
8586
} else {
8687
pipelineRunner = new PortablePipelineJarCreator(FlinkPipelineRunner.class);
8788
}
8889

89-
flinkOptions.setRunner(null);
90-
9190
LOG.info("Invoking job {} with pipeline runner {}", invocationId, pipelineRunner);
9291
return createJobInvocation(
9392
invocationId, retrievalToken, executorService, pipeline, flinkOptions, pipelineRunner);

runners/java-job-service/src/main/java/org/apache/beam/runners/jobsubmission/JobInvoker.java

+2-3
Original file line numberDiff line numberDiff line change
@@ -20,7 +20,6 @@
2020
import java.io.IOException;
2121
import java.util.concurrent.Executors;
2222
import java.util.concurrent.ThreadFactory;
23-
import javax.annotation.Nullable;
2423
import org.apache.beam.model.pipeline.v1.RunnerApi;
2524
import org.apache.beam.vendor.grpc.v1p60p1.com.google.protobuf.Struct;
2625
import org.apache.beam.vendor.guava.v32_1_2_jre.com.google.common.util.concurrent.ListeningExecutorService;
@@ -39,11 +38,11 @@ public abstract class JobInvoker {
3938
protected abstract JobInvocation invokeWithExecutor(
4039
RunnerApi.Pipeline pipeline,
4140
Struct options,
42-
@Nullable String retrievalToken,
41+
String retrievalToken,
4342
ListeningExecutorService executorService)
4443
throws IOException;
4544

46-
JobInvocation invoke(RunnerApi.Pipeline pipeline, Struct options, @Nullable String retrievalToken)
45+
JobInvocation invoke(RunnerApi.Pipeline pipeline, Struct options, String retrievalToken)
4746
throws IOException {
4847
return invokeWithExecutor(pipeline, options, retrievalToken, this.executorService);
4948
}

0 commit comments

Comments
 (0)