public abstract class Dispatcher extends PermanentlyFencedRpcEndpoint<DispatcherId> implements DispatcherGateway
RpcEndpoint.MainThreadExecutor
Modifier and Type | Field and Description |
---|---|
static String |
DISPATCHER_NAME |
protected CompletableFuture<ApplicationStatus> |
shutDownFuture |
log, rpcServer
Constructor and Description |
---|
Dispatcher(RpcService rpcService,
DispatcherId fencingToken,
Collection<JobGraph> recoveredJobs,
DispatcherBootstrapFactory dispatcherBootstrapFactory,
DispatcherServices dispatcherServices) |
Modifier and Type | Method and Description |
---|---|
CompletableFuture<Acknowledge> |
cancelJob(JobID jobId,
Time timeout)
Cancel the given job.
|
CompletableFuture<CoordinationResponse> |
deliverCoordinationRequestToCoordinator(JobID jobId,
OperatorID operatorId,
SerializedValue<CoordinationRequest> serializedRequest,
Time timeout)
Deliver a coordination request to a specified coordinator and return the response.
|
CompletableFuture<Acknowledge> |
disposeSavepoint(String savepointPath,
Time timeout)
Dispose the given savepoint.
|
CompletableFuture<Integer> |
getBlobServerPort(Time timeout)
Returns the port of the blob server.
|
CompletableFuture<ApplicationStatus> |
getShutDownFuture() |
protected org.apache.flink.runtime.dispatcher.Dispatcher.CleanupJobState |
jobNotFinished(JobID jobId) |
protected org.apache.flink.runtime.dispatcher.Dispatcher.CleanupJobState |
jobReachedGloballyTerminalState(ArchivedExecutionGraph archivedExecutionGraph) |
CompletableFuture<Collection<JobID>> |
listJobs(Time timeout)
List the current set of submitted jobs.
|
protected void |
onFatalError(Throwable throwable) |
CompletableFuture<Void> |
onRemovedJobGraph(JobID jobId) |
void |
onStart()
User overridable callback which is called from
RpcEndpoint.internalCallOnStart() . |
CompletableFuture<Void> |
onStop()
User overridable callback which is called from
RpcEndpoint.internalCallOnStop() . |
CompletableFuture<ClusterOverview> |
requestClusterOverview(Time timeout)
Requests the cluster status overview.
|
CompletableFuture<ArchivedExecutionGraph> |
requestJob(JobID jobId,
Time timeout)
Requests the
ArchivedExecutionGraph for the given jobId. |
CompletableFuture<JobResult> |
requestJobResult(JobID jobId,
Time timeout)
Requests the
JobResult of a job specified by the given jobId. |
CompletableFuture<JobStatus> |
requestJobStatus(JobID jobId,
Time timeout)
Request the
JobStatus of the given job. |
CompletableFuture<Collection<String>> |
requestMetricQueryServiceAddresses(Time timeout)
Requests the addresses of the
MetricQueryService to query. |
CompletableFuture<MultipleJobsDetails> |
requestMultipleJobDetails(Time timeout)
Requests job details currently being executed on the Flink cluster.
|
CompletableFuture<OperatorBackPressureStatsResponse> |
requestOperatorBackPressureStats(JobID jobId,
JobVertexID jobVertexId)
Requests the statistics on operator back pressure.
|
CompletableFuture<Collection<Tuple2<ResourceID,String>>> |
requestTaskManagerMetricQueryServiceAddresses(Time timeout)
Requests the addresses for the TaskManagers'
MetricQueryService to query. |
CompletableFuture<Acknowledge> |
shutDownCluster() |
CompletableFuture<Acknowledge> |
shutDownCluster(ApplicationStatus applicationStatus) |
CompletableFuture<String> |
stopWithSavepoint(JobID jobId,
String targetDirectory,
boolean terminate,
Time timeout)
Stops the job with a savepoint.
|
CompletableFuture<Acknowledge> |
submitJob(JobGraph jobGraph,
Time timeout)
Submit a job to the dispatcher.
|
CompletableFuture<String> |
triggerSavepoint(JobID jobId,
String targetDirectory,
boolean cancelJob,
Time timeout)
Triggers a savepoint with the given savepoint directory as a target.
|
setFencingToken
callAsyncWithoutFencing, getFencingToken, getMainThreadExecutor, getUnfencedMainThreadExecutor, runAsyncWithoutFencing
callAsync, closeAsync, getAddress, getEndpointId, getHostname, getRpcService, getSelfGateway, getTerminationFuture, internalCallOnStart, internalCallOnStop, isRunning, runAsync, scheduleRunAsync, scheduleRunAsync, start, stop, validateRunsInMainThread
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
getFencingToken
getAddress, getHostname
close
public static final String DISPATCHER_NAME
protected final CompletableFuture<ApplicationStatus> shutDownFuture
public Dispatcher(RpcService rpcService, DispatcherId fencingToken, Collection<JobGraph> recoveredJobs, DispatcherBootstrapFactory dispatcherBootstrapFactory, DispatcherServices dispatcherServices) throws Exception
Exception
public CompletableFuture<ApplicationStatus> getShutDownFuture()
public void onStart() throws Exception
RpcEndpoint
RpcEndpoint.internalCallOnStart()
.
This method is called when the RpcEndpoint is being started. The method is guaranteed to be executed in the main thread context and can be used to start the rpc endpoint in the context of the rpc endpoint's main thread.
IMPORTANT: This method should never be called directly by the user.
onStart
in class RpcEndpoint
Exception
- indicating that the rpc endpoint could not be started. If an exception
occurs, then the rpc endpoint will automatically terminate.public CompletableFuture<Void> onStop()
RpcEndpoint
RpcEndpoint.internalCallOnStop()
.
This method is called when the RpcEndpoint is being shut down. The method is guaranteed to be executed in the main thread context and can be used to clean up internal state.
IMPORTANT: This method should never be called directly by the user.
onStop
in class RpcEndpoint
public CompletableFuture<Acknowledge> submitJob(JobGraph jobGraph, Time timeout)
DispatcherGateway
submitJob
in interface DispatcherGateway
jobGraph
- JobGraph to submittimeout
- RPC timeoutpublic CompletableFuture<Collection<JobID>> listJobs(Time timeout)
DispatcherGateway
listJobs
in interface DispatcherGateway
timeout
- RPC timeoutpublic CompletableFuture<Acknowledge> disposeSavepoint(String savepointPath, Time timeout)
RestfulGateway
disposeSavepoint
in interface RestfulGateway
savepointPath
- identifying the savepoint to disposetimeout
- RPC timeoutpublic CompletableFuture<Acknowledge> cancelJob(JobID jobId, Time timeout)
RestfulGateway
cancelJob
in interface RestfulGateway
jobId
- identifying the job to canceltimeout
- of the operationpublic CompletableFuture<ClusterOverview> requestClusterOverview(Time timeout)
RestfulGateway
requestClusterOverview
in interface RestfulGateway
timeout
- for the asynchronous operationpublic CompletableFuture<MultipleJobsDetails> requestMultipleJobDetails(Time timeout)
RestfulGateway
requestMultipleJobDetails
in interface RestfulGateway
timeout
- for the asynchronous operationpublic CompletableFuture<JobStatus> requestJobStatus(JobID jobId, Time timeout)
RestfulGateway
JobStatus
of the given job.requestJobStatus
in interface RestfulGateway
jobId
- identifying the job for which to retrieve the JobStatustimeout
- for the asynchronous operationJobStatus
of the given jobpublic CompletableFuture<OperatorBackPressureStatsResponse> requestOperatorBackPressureStats(JobID jobId, JobVertexID jobVertexId)
RestfulGateway
requestOperatorBackPressureStats
in interface RestfulGateway
jobId
- Job for which the stats are requested.jobVertexId
- JobVertex for which the stats are requested.OperatorBackPressureStatsResponse
.public CompletableFuture<ArchivedExecutionGraph> requestJob(JobID jobId, Time timeout)
DispatcherGateway
ArchivedExecutionGraph
for the given jobId. If there is no such graph,
then the future is completed with a FlinkJobNotFoundException
.
Note: We enforce that the returned future contains a ArchivedExecutionGraph
unlike
the super interface.
requestJob
in interface DispatcherGateway
requestJob
in interface RestfulGateway
jobId
- identifying the job whose AccessExecutionGraph is requestedtimeout
- for the asynchronous operationFlinkJobNotFoundException
public CompletableFuture<JobResult> requestJobResult(JobID jobId, Time timeout)
RestfulGateway
JobResult
of a job specified by the given jobId.requestJobResult
in interface RestfulGateway
jobId
- identifying the job for which to retrieve the JobResult
.timeout
- for the asynchronous operationJobResult
once the job has finishedpublic CompletableFuture<Collection<String>> requestMetricQueryServiceAddresses(Time timeout)
RestfulGateway
MetricQueryService
to query.requestMetricQueryServiceAddresses
in interface RestfulGateway
timeout
- for the asynchronous operationpublic CompletableFuture<Collection<Tuple2<ResourceID,String>>> requestTaskManagerMetricQueryServiceAddresses(Time timeout)
RestfulGateway
MetricQueryService
to query.requestTaskManagerMetricQueryServiceAddresses
in interface RestfulGateway
timeout
- for the asynchronous operationpublic CompletableFuture<Integer> getBlobServerPort(Time timeout)
DispatcherGateway
getBlobServerPort
in interface DispatcherGateway
timeout
- of the operationpublic CompletableFuture<String> triggerSavepoint(JobID jobId, String targetDirectory, boolean cancelJob, Time timeout)
RestfulGateway
triggerSavepoint
in interface RestfulGateway
jobId
- ID of the job for which the savepoint should be triggered.targetDirectory
- Target directory for the savepoint.timeout
- Timeout for the asynchronous operationexternal pointer
of
the savepoint.public CompletableFuture<String> stopWithSavepoint(JobID jobId, String targetDirectory, boolean terminate, Time timeout)
RestfulGateway
stopWithSavepoint
in interface RestfulGateway
jobId
- ID of the job for which the savepoint should be triggered.targetDirectory
- to which to write the savepoint data or null if the default savepoint
directory should be usedterminate
- flag indicating if the job should terminate or just suspendtimeout
- for the rpc callpublic CompletableFuture<Acknowledge> shutDownCluster()
shutDownCluster
in interface RestfulGateway
public CompletableFuture<Acknowledge> shutDownCluster(ApplicationStatus applicationStatus)
shutDownCluster
in interface DispatcherGateway
public CompletableFuture<CoordinationResponse> deliverCoordinationRequestToCoordinator(JobID jobId, OperatorID operatorId, SerializedValue<CoordinationRequest> serializedRequest, Time timeout)
RestfulGateway
deliverCoordinationRequestToCoordinator
in interface RestfulGateway
jobId
- identifying the job which the coordinator belongs tooperatorId
- identifying the coordinator to receive the requestserializedRequest
- serialized request to delivertimeout
- RPC timeoutFlinkException
if the task is not running, or no
operator/coordinator exists for the given ID, or the coordinator cannot handle client
events.protected void onFatalError(Throwable throwable)
protected org.apache.flink.runtime.dispatcher.Dispatcher.CleanupJobState jobReachedGloballyTerminalState(ArchivedExecutionGraph archivedExecutionGraph)
protected org.apache.flink.runtime.dispatcher.Dispatcher.CleanupJobState jobNotFinished(JobID jobId)
public CompletableFuture<Void> onRemovedJobGraph(JobID jobId)
Copyright © 2014–2021 The Apache Software Foundation. All rights reserved.