Skip to content

Commit a699e8e

Browse files
authored
Fix The PostCommit XVR Flink job (#34628)
* Revert PR #33713 * Run SpotlessJavaApply
1 parent 5a590ad commit a699e8e

File tree

2 files changed

+12
-10
lines changed

2 files changed

+12
-10
lines changed

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

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

20-
import static org.apache.beam.sdk.util.Preconditions.checkStateNotNull;
2120
import static org.apache.beam.sdk.util.construction.resources.PipelineResources.detectClassPathResourcesToStage;
2221

2322
import java.util.UUID;
@@ -32,10 +31,14 @@
3231
import org.apache.beam.vendor.grpc.v1p69p0.com.google.protobuf.Struct;
3332
import org.apache.beam.vendor.guava.v32_1_2_jre.com.google.common.base.Strings;
3433
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+
})
3942
public class FlinkJobInvoker extends JobInvoker {
4043
private static final Logger LOG = LoggerFactory.getLogger(FlinkJobInvoker.class);
4144

@@ -54,7 +57,7 @@ protected FlinkJobInvoker(FlinkJobServerDriver.FlinkServerConfiguration serverCo
5457
protected JobInvocation invokeWithExecutor(
5558
RunnerApi.Pipeline pipeline,
5659
Struct options,
57-
String retrievalToken,
60+
@Nullable String retrievalToken,
5861
ListeningExecutorService executorService) {
5962

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

7275
PortablePipelineOptions portableOptions = flinkOptions.as(PortablePipelineOptions.class);
7376

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-
7977
PortablePipelineRunner pipelineRunner;
8078
if (Strings.isNullOrEmpty(portableOptions.getOutputExecutablePath())) {
8179
pipelineRunner =
8280
new FlinkPipelineRunner(
8381
flinkOptions,
8482
serverConfig.getFlinkConfDir(),
85-
detectClassPathResourcesToStage(thisClassLoader, flinkOptions));
83+
detectClassPathResourcesToStage(
84+
FlinkJobInvoker.class.getClassLoader(), flinkOptions));
8685
} else {
8786
pipelineRunner = new PortablePipelineJarCreator(FlinkPipelineRunner.class);
8887
}
8988

89+
flinkOptions.setRunner(null);
90+
9091
LOG.info("Invoking job {} with pipeline runner {}", invocationId, pipelineRunner);
9192
return createJobInvocation(
9293
invocationId, retrievalToken, executorService, pipeline, flinkOptions, pipelineRunner);

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

Lines changed: 3 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -20,6 +20,7 @@
2020
import java.io.IOException;
2121
import java.util.concurrent.Executors;
2222
import java.util.concurrent.ThreadFactory;
23+
import javax.annotation.Nullable;
2324
import org.apache.beam.model.pipeline.v1.RunnerApi;
2425
import org.apache.beam.vendor.grpc.v1p69p0.com.google.protobuf.Struct;
2526
import org.apache.beam.vendor.guava.v32_1_2_jre.com.google.common.util.concurrent.ListeningExecutorService;
@@ -38,11 +39,11 @@ public abstract class JobInvoker {
3839
protected abstract JobInvocation invokeWithExecutor(
3940
RunnerApi.Pipeline pipeline,
4041
Struct options,
41-
String retrievalToken,
42+
@Nullable String retrievalToken,
4243
ListeningExecutorService executorService)
4344
throws IOException;
4445

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

0 commit comments

Comments
 (0)