diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/dispatcher/Dispatcher.java b/flink-runtime/src/main/java/org/apache/flink/runtime/dispatcher/Dispatcher.java index b22e7abe1fd55..1fa0f7e8b3292 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/dispatcher/Dispatcher.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/dispatcher/Dispatcher.java @@ -56,13 +56,14 @@ import org.apache.flink.util.FlinkException; import org.apache.flink.util.Preconditions; +import javax.annotation.Nullable; + import java.io.IOException; import java.util.ArrayList; import java.util.Collection; import java.util.Collections; import java.util.HashMap; import java.util.Map; -import java.util.Optional; import java.util.UUID; import java.util.concurrent.CompletableFuture; @@ -93,7 +94,8 @@ public abstract class Dispatcher extends FencedRpcEndpoint impleme private final LeaderElectionService leaderElectionService; - private final CompletableFuture restAddressFuture; + @Nullable + protected final String restAddress; protected Dispatcher( RpcService rpcService, @@ -105,7 +107,7 @@ protected Dispatcher( HeartbeatServices heartbeatServices, MetricRegistry metricRegistry, FatalErrorHandler fatalErrorHandler, - Optional restAddress) throws Exception { + @Nullable String restAddress) throws Exception { super(rpcService, endpointId); this.configuration = Preconditions.checkNotNull(configuration); @@ -124,10 +126,8 @@ protected Dispatcher( jobManagerRunners = new HashMap<>(16); leaderElectionService = highAvailabilityServices.getDispatcherLeaderElectionService(); - this.restAddressFuture = restAddress - .map(CompletableFuture::completedFuture) - .orElse(FutureUtils.completedExceptionally(new DispatcherException("The Dispatcher has not been started with a REST endpoint."))); + this.restAddress = restAddress; } //------------------------------------------------------ @@ -275,7 +275,11 @@ public CompletableFuture stopJob(JobID jobId, Time timeout) { @Override public CompletableFuture requestRestAddress(Time timeout) { - return restAddressFuture; + if (restAddress != null) { + return CompletableFuture.completedFuture(restAddress); + } else { + return FutureUtils.completedExceptionally(new DispatcherException("The Dispatcher has not been started with a REST endpoint.")); + } } @Override diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/dispatcher/StandaloneDispatcher.java b/flink-runtime/src/main/java/org/apache/flink/runtime/dispatcher/StandaloneDispatcher.java index 5a6889ef36aae..3ba681cfb64ce 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/dispatcher/StandaloneDispatcher.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/dispatcher/StandaloneDispatcher.java @@ -33,7 +33,7 @@ import org.apache.flink.runtime.rpc.FatalErrorHandler; import org.apache.flink.runtime.rpc.RpcService; -import java.util.Optional; +import javax.annotation.Nullable; /** * Dispatcher implementation which spawns a {@link JobMaster} for each @@ -51,7 +51,7 @@ public StandaloneDispatcher( HeartbeatServices heartbeatServices, MetricRegistry metricRegistry, FatalErrorHandler fatalErrorHandler, - Optional restAddress) throws Exception { + @Nullable String restAddress) throws Exception { super( rpcService, endpointId, diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/entrypoint/SessionClusterEntrypoint.java b/flink-runtime/src/main/java/org/apache/flink/runtime/entrypoint/SessionClusterEntrypoint.java index 3feb0055e38f0..27ddf49a2e5ab 100644 --- a/flink-runtime/src/main/java/org/apache/flink/runtime/entrypoint/SessionClusterEntrypoint.java +++ b/flink-runtime/src/main/java/org/apache/flink/runtime/entrypoint/SessionClusterEntrypoint.java @@ -49,7 +49,8 @@ import akka.actor.ActorSystem; -import java.util.Optional; +import javax.annotation.Nullable; + import java.util.concurrent.Executor; /** @@ -130,7 +131,7 @@ protected void startClusterComponents( heartbeatServices, metricRegistry, this, - Optional.of(dispatcherRestEndpoint.getRestAddress())); + dispatcherRestEndpoint.getRestAddress()); LOG.debug("Starting ResourceManager."); resourceManager.start(); @@ -214,7 +215,7 @@ protected Dispatcher createDispatcher( HeartbeatServices heartbeatServices, MetricRegistry metricRegistry, FatalErrorHandler fatalErrorHandler, - Optional restAddress) throws Exception { + @Nullable String restAddress) throws Exception { // create the default dispatcher return new StandaloneDispatcher( diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/dispatcher/DispatcherTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/dispatcher/DispatcherTest.java index a511d455b7e25..d5b63d4f615ce 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/dispatcher/DispatcherTest.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/dispatcher/DispatcherTest.java @@ -51,7 +51,6 @@ import org.junit.rules.TestName; import org.mockito.Mockito; -import java.util.Optional; import java.util.UUID; import java.util.concurrent.CompletableFuture; import java.util.concurrent.TimeUnit; @@ -224,7 +223,7 @@ protected TestingDispatcher( heartbeatServices, metricRegistry, fatalErrorHandler, - Optional.empty()); + null); this.jobManagerRunner = jobManagerRunner; this.expectedJobId = expectedJobId;