public abstract class Dispatcher extends FencedRpcEndpoint<DispatcherId> implements DispatcherGateway, LeaderContender, SubmittedJobGraphStore.SubmittedJobGraphListener
RpcEndpoint.MainThreadExecutor
Modifier and Type | Field and Description |
---|---|
static String |
DISPATCHER_NAME |
log, rpcServer
Constructor and Description |
---|
Dispatcher(RpcService rpcService,
String endpointId,
Configuration configuration,
HighAvailabilityServices highAvailabilityServices,
SubmittedJobGraphStore submittedJobGraphStore,
GatewayRetriever<ResourceManagerGateway> resourceManagerGatewayRetriever,
BlobServer blobServer,
HeartbeatServices heartbeatServices,
JobManagerMetricGroup jobManagerMetricGroup,
String metricServiceQueryPath,
ArchivedExecutionGraphStore archivedExecutionGraphStore,
JobManagerRunnerFactory jobManagerRunnerFactory,
FatalErrorHandler fatalErrorHandler,
HistoryServerArchivist historyServerArchivist) |
Modifier and Type | Method and Description |
---|---|
CompletableFuture<Acknowledge> |
cancelJob(JobID jobId,
Time timeout)
Cancel the given job.
|
CompletableFuture<Acknowledge> |
disposeSavepoint(String savepointPath,
Time timeout)
Dispose the given savepoint.
|
CompletableFuture<Integer> |
getBlobServerPort(Time timeout)
Returns the port of the blob server.
|
void |
grantLeadership(UUID newLeaderSessionID)
Callback method when current resourceManager is granted leadership.
|
void |
handleError(Exception exception)
Handles error occurring in the leader election service.
|
protected void |
jobNotFinished(JobID jobId) |
protected void |
jobReachedGloballyTerminalState(ArchivedExecutionGraph archivedExecutionGraph) |
CompletableFuture<Collection<JobID>> |
listJobs(Time timeout)
List the current set of submitted jobs.
|
void |
onAddedJobGraph(JobID jobId)
Callback for
SubmittedJobGraph instances added by a different SubmittedJobGraphStore instance. |
protected void |
onFatalError(Throwable throwable) |
void |
onRemovedJobGraph(JobID jobId)
Callback for
SubmittedJobGraph instances removed by a different SubmittedJobGraphStore instance. |
void |
onStart()
User overridable callback.
|
CompletableFuture<Void> |
onStop()
User overridable callback.
|
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>> |
requestMetricQueryServicePaths(Time timeout)
Requests the paths for 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>>> |
requestTaskManagerMetricQueryServicePaths(Time timeout)
Requests the paths for the TaskManager's
MetricQueryService to query. |
CompletableFuture<Acknowledge> |
rescaleJob(JobID jobId,
int newParallelism,
RescalingBehaviour rescalingBehaviour,
Time timeout)
Trigger rescaling of the given job.
|
void |
revokeLeadership()
Callback method when current resourceManager loses leadership.
|
CompletableFuture<Acknowledge> |
shutDownCluster() |
CompletableFuture<Acknowledge> |
stopJob(JobID jobId,
Time timeout)
Stop the given job.
|
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.
|
callAsyncWithoutFencing, getFencingToken, getMainThreadExecutor, getUnfencedMainThreadExecutor, runAsyncWithoutFencing, setFencingToken
callAsync, closeAsync, getAddress, getEndpointId, getHostname, getRpcService, getSelfGateway, getTerminationFuture, runAsync, scheduleRunAsync, scheduleRunAsync, start, stop, validateRunsInMainThread
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
getFencingToken
getAddress, getHostname
getAddress
close
public static final String DISPATCHER_NAME
public Dispatcher(RpcService rpcService, String endpointId, Configuration configuration, HighAvailabilityServices highAvailabilityServices, SubmittedJobGraphStore submittedJobGraphStore, GatewayRetriever<ResourceManagerGateway> resourceManagerGatewayRetriever, BlobServer blobServer, HeartbeatServices heartbeatServices, JobManagerMetricGroup jobManagerMetricGroup, @Nullable String metricServiceQueryPath, ArchivedExecutionGraphStore archivedExecutionGraphStore, JobManagerRunnerFactory jobManagerRunnerFactory, FatalErrorHandler fatalErrorHandler, HistoryServerArchivist historyServerArchivist) throws Exception
Exception
public void onStart() throws Exception
RpcEndpoint
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
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<Acknowledge> stopJob(JobID jobId, Time timeout)
RestfulGateway
stopJob
in interface RestfulGateway
jobId
- identifying the job to stoptimeout
- of the operationpublic CompletableFuture<Acknowledge> rescaleJob(JobID jobId, int newParallelism, RescalingBehaviour rescalingBehaviour, Time timeout)
RestfulGateway
rescaleJob
in interface RestfulGateway
jobId
- specifying the job to rescalenewParallelism
- new parallelism of the jobrescalingBehaviour
- defining how strict the rescaling has to be executedtimeout
- of this operationAcknowledge
once the rescaling was successfulpublic 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>> requestMetricQueryServicePaths(Time timeout)
RestfulGateway
MetricQueryService
to query.requestMetricQueryServicePaths
in interface RestfulGateway
timeout
- for the asynchronous operationpublic CompletableFuture<Collection<Tuple2<ResourceID,String>>> requestTaskManagerMetricQueryServicePaths(Time timeout)
RestfulGateway
MetricQueryService
to query.requestTaskManagerMetricQueryServicePaths
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<Acknowledge> shutDownCluster()
shutDownCluster
in interface RestfulGateway
protected void onFatalError(Throwable throwable)
protected void jobReachedGloballyTerminalState(ArchivedExecutionGraph archivedExecutionGraph)
protected void jobNotFinished(JobID jobId)
public void grantLeadership(UUID newLeaderSessionID)
grantLeadership
in interface LeaderContender
newLeaderSessionID
- unique leadershipIDpublic void revokeLeadership()
revokeLeadership
in interface LeaderContender
public void handleError(Exception exception)
handleError
in interface LeaderContender
exception
- Exception being thrown in the leader election servicepublic void onAddedJobGraph(JobID jobId)
SubmittedJobGraphStore.SubmittedJobGraphListener
SubmittedJobGraph
instances added by a different SubmittedJobGraphStore
instance.
Important: It is possible to get false positives and be notified about a job graph, which was added by this instance.
onAddedJobGraph
in interface SubmittedJobGraphStore.SubmittedJobGraphListener
jobId
- The JobID
of the added job graphpublic void onRemovedJobGraph(JobID jobId)
SubmittedJobGraphStore.SubmittedJobGraphListener
SubmittedJobGraph
instances removed by a different SubmittedJobGraphStore
instance.onRemovedJobGraph
in interface SubmittedJobGraphStore.SubmittedJobGraphListener
jobId
- The JobID
of the removed job graphCopyright © 2014–2020 The Apache Software Foundation. All rights reserved.