Class Dispatcher
- java.lang.Object
-
- org.apache.flink.runtime.rpc.RpcEndpoint
-
- org.apache.flink.runtime.rpc.FencedRpcEndpoint<DispatcherId>
-
- org.apache.flink.runtime.dispatcher.Dispatcher
-
- All Implemented Interfaces:
AutoCloseable
,DispatcherGateway
,FencedRpcGateway<DispatcherId>
,RpcGateway
,RestfulGateway
,AutoCloseableAsync
- Direct Known Subclasses:
MiniDispatcher
,StandaloneDispatcher
public abstract class Dispatcher extends FencedRpcEndpoint<DispatcherId> implements DispatcherGateway
Base class for the Dispatcher component. The Dispatcher component is responsible for receiving job submissions, persisting them, spawning JobManagers to execute the jobs and to recover them in case of a master failure. Furthermore, it knows about the state of the Flink session cluster.
-
-
Nested Class Summary
Nested Classes Modifier and Type Class Description protected static class
Dispatcher.ExecutionType
Enum to distinguish between initial job submission and re-submission for recovery.-
Nested classes/interfaces inherited from class org.apache.flink.runtime.rpc.RpcEndpoint
RpcEndpoint.MainThreadExecutor
-
-
Field Summary
Fields Modifier and Type Field Description static ConfigOption<Duration>
CLIENT_ALIVENESS_CHECK_DURATION
static String
DISPATCHER_NAME
protected CompletableFuture<ApplicationStatus>
shutDownFuture
-
Fields inherited from class org.apache.flink.runtime.rpc.RpcEndpoint
log, rpcServer
-
-
Constructor Summary
Constructors Modifier Constructor Description Dispatcher(RpcService rpcService, DispatcherId fencingToken, Collection<ExecutionPlan> recoveredJobs, Collection<JobResult> recoveredDirtyJobs, DispatcherBootstrapFactory dispatcherBootstrapFactory, DispatcherServices dispatcherServices)
protected
Dispatcher(RpcService rpcService, DispatcherId fencingToken, Collection<ExecutionPlan> recoveredJobs, Collection<JobResult> recoveredDirtyJobs, DispatcherBootstrapFactory dispatcherBootstrapFactory, DispatcherServices dispatcherServices, JobManagerRunnerRegistry jobManagerRunnerRegistry, ResourceCleanerFactory resourceCleanerFactory)
-
Method Summary
All Methods Instance Methods Concrete Methods Modifier and Type Method Description CompletableFuture<Acknowledge>
cancelJob(JobID jobId, Duration timeout)
Cancel the given job.CompletableFuture<CoordinationResponse>
deliverCoordinationRequestToCoordinator(JobID jobId, String operatorUid, SerializedValue<CoordinationRequest> serializedRequest, Duration timeout)
Deliver a coordination request to a specified coordinator and return the response.CompletableFuture<Acknowledge>
disposeSavepoint(String savepointPath, Duration timeout)
Dispose the given savepoint.CompletableFuture<Integer>
getBlobServerPort(Duration timeout)
Returns the port of the blob server.CompletableFuture<ApplicationStatus>
getShutDownFuture()
CompletableFuture<OperationResult<Long>>
getTriggeredCheckpointStatus(AsynchronousJobOperationKey operationKey)
Get the status of a checkpoint triggered under the specified operation key.CompletableFuture<OperationResult<String>>
getTriggeredSavepointStatus(AsynchronousJobOperationKey operationKey)
Get the status of a savepoint triggered under the specified operation key.protected CompletableFuture<org.apache.flink.runtime.dispatcher.Dispatcher.CleanupJobState>
jobReachedTerminalState(ExecutionGraphInfo executionGraphInfo)
CompletableFuture<Collection<JobID>>
listJobs(Duration timeout)
List the current set of submitted jobs.protected void
onFatalError(Throwable throwable)
CompletableFuture<Void>
onRemovedExecutionPlan(JobID jobId)
void
onStart()
User overridable callback which is called fromRpcEndpoint.internalCallOnStart()
.CompletableFuture<Void>
onStop()
User overridable callback which is called fromRpcEndpoint.internalCallOnStop()
.CompletableFuture<Void>
reportJobClientHeartbeat(JobID jobId, long expiredTimestamp, Duration timeout)
The client reports the heartbeat to the dispatcher for aliveness.CompletableFuture<CheckpointStatsSnapshot>
requestCheckpointStats(JobID jobId, Duration timeout)
Requests theCheckpointStatsSnapshot
containing checkpointing information.CompletableFuture<ClusterOverview>
requestClusterOverview(Duration timeout)
Requests the cluster status overview.CompletableFuture<ExecutionGraphInfo>
requestExecutionGraphInfo(JobID jobId, Duration timeout)
Requests theExecutionGraphInfo
containing additional information besides theArchivedExecutionGraph
.CompletableFuture<JobResourceRequirements>
requestJobResourceRequirements(JobID jobId)
Read currentjob resource requirements
for a given job.CompletableFuture<JobResult>
requestJobResult(JobID jobId, Duration timeout)
Requests theJobResult
of a job specified by the given jobId.CompletableFuture<JobStatus>
requestJobStatus(JobID jobId, Duration timeout)
Request theJobStatus
of the given job.CompletableFuture<Collection<String>>
requestMetricQueryServiceAddresses(Duration timeout)
Requests the addresses of theMetricQueryService
to query.CompletableFuture<MultipleJobsDetails>
requestMultipleJobDetails(Duration timeout)
Requests job details currently being executed on the Flink cluster.CompletableFuture<Collection<Tuple2<ResourceID,String>>>
requestTaskManagerMetricQueryServiceAddresses(Duration timeout)
Requests the addresses for the TaskManagers'MetricQueryService
to query.CompletableFuture<ThreadDumpInfo>
requestThreadDump(Duration timeout)
Requests the thread dump from the JobManager.protected void
runPostJobGloballyTerminated(JobID jobId, JobStatus jobStatus)
CompletableFuture<Acknowledge>
shutDownCluster()
CompletableFuture<Acknowledge>
shutDownCluster(ApplicationStatus applicationStatus)
CompletableFuture<Acknowledge>
stopWithSavepoint(AsynchronousJobOperationKey operationKey, String targetDirectory, SavepointFormatType formatType, TriggerSavepointMode savepointMode, Duration timeout)
Stops the job with a savepoint, returning a future that completes when the operation is started.CompletableFuture<String>
stopWithSavepointAndGetLocation(JobID jobId, String targetDirectory, SavepointFormatType formatType, TriggerSavepointMode savepointMode, Duration timeout)
Stops the job with a savepoint, returning a future that completes with the savepoint location when the savepoint is completed.CompletableFuture<Acknowledge>
submitFailedJob(JobID jobId, String jobName, Throwable exception)
CompletableFuture<Acknowledge>
submitJob(ExecutionPlan executionPlan, Duration timeout)
Submit a job to the dispatcher.CompletableFuture<String>
triggerCheckpoint(JobID jobID, Duration timeout)
CompletableFuture<Acknowledge>
triggerCheckpoint(AsynchronousJobOperationKey operationKey, CheckpointType checkpointType, Duration timeout)
Triggers a checkpoint with the given savepoint directory as a target.CompletableFuture<Long>
triggerCheckpointAndGetCheckpointID(JobID jobID, CheckpointType checkpointType, Duration timeout)
Triggers a checkpoint, returning a future that completes with the checkpoint id when it is complete.CompletableFuture<Acknowledge>
triggerSavepoint(AsynchronousJobOperationKey operationKey, String targetDirectory, SavepointFormatType formatType, TriggerSavepointMode savepointMode, Duration timeout)
Triggers a savepoint with the given savepoint directory as a target, returning a future that completes when the operation is started.CompletableFuture<String>
triggerSavepointAndGetLocation(JobID jobId, String targetDirectory, SavepointFormatType formatType, TriggerSavepointMode savepointMode, Duration timeout)
Triggers a savepoint with the given savepoint directory as a target, returning a future that completes with the savepoint location when it is complete.CompletableFuture<Acknowledge>
updateJobResourceRequirements(JobID jobId, JobResourceRequirements jobResourceRequirements)
Updatejob resource requirements
for a given job.-
Methods inherited from class org.apache.flink.runtime.rpc.FencedRpcEndpoint
getFencingToken
-
Methods inherited from class org.apache.flink.runtime.rpc.RpcEndpoint
callAsync, closeAsync, getAddress, getEndpointId, getHostname, getMainThreadExecutor, getMainThreadExecutor, getRpcService, getSelfGateway, getTerminationFuture, internalCallOnStart, internalCallOnStop, isRunning, registerResource, runAsync, scheduleRunAsync, scheduleRunAsync, start, stop, unregisterResource, validateRunsInMainThread
-
Methods inherited from class java.lang.Object
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
-
Methods inherited from interface org.apache.flink.util.AutoCloseableAsync
close
-
Methods inherited from interface org.apache.flink.runtime.rpc.FencedRpcGateway
getFencingToken
-
Methods inherited from interface org.apache.flink.runtime.webmonitor.RestfulGateway
requestJob
-
Methods inherited from interface org.apache.flink.runtime.rpc.RpcGateway
getAddress, getHostname
-
-
-
-
Field Detail
-
CLIENT_ALIVENESS_CHECK_DURATION
@VisibleForTesting @Internal public static final ConfigOption<Duration> CLIENT_ALIVENESS_CHECK_DURATION
-
DISPATCHER_NAME
public static final String DISPATCHER_NAME
- See Also:
- Constant Field Values
-
shutDownFuture
protected final CompletableFuture<ApplicationStatus> shutDownFuture
-
-
Constructor Detail
-
Dispatcher
public Dispatcher(RpcService rpcService, DispatcherId fencingToken, Collection<ExecutionPlan> recoveredJobs, Collection<JobResult> recoveredDirtyJobs, DispatcherBootstrapFactory dispatcherBootstrapFactory, DispatcherServices dispatcherServices) throws Exception
- Throws:
Exception
-
Dispatcher
@VisibleForTesting protected Dispatcher(RpcService rpcService, DispatcherId fencingToken, Collection<ExecutionPlan> recoveredJobs, Collection<JobResult> recoveredDirtyJobs, DispatcherBootstrapFactory dispatcherBootstrapFactory, DispatcherServices dispatcherServices, JobManagerRunnerRegistry jobManagerRunnerRegistry, ResourceCleanerFactory resourceCleanerFactory) throws Exception
- Throws:
Exception
-
-
Method Detail
-
getShutDownFuture
public CompletableFuture<ApplicationStatus> getShutDownFuture()
-
onStart
public void onStart() throws Exception
Description copied from class:RpcEndpoint
User overridable callback which is called fromRpcEndpoint.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.
- Overrides:
onStart
in classRpcEndpoint
- Throws:
Exception
- indicating that the rpc endpoint could not be started. If an exception occurs, then the rpc endpoint will automatically terminate.
-
onStop
public CompletableFuture<Void> onStop()
Description copied from class:RpcEndpoint
User overridable callback which is called fromRpcEndpoint.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.
- Overrides:
onStop
in classRpcEndpoint
- Returns:
- Future which is completed once all post stop actions are completed. If an error occurs this future is completed exceptionally
-
submitJob
public CompletableFuture<Acknowledge> submitJob(ExecutionPlan executionPlan, Duration timeout)
Description copied from interface:DispatcherGateway
Submit a job to the dispatcher.- Specified by:
submitJob
in interfaceDispatcherGateway
- Parameters:
executionPlan
- ExecutionPlan to submittimeout
- RPC timeout- Returns:
- A future acknowledge if the submission succeeded
-
submitFailedJob
public CompletableFuture<Acknowledge> submitFailedJob(JobID jobId, String jobName, Throwable exception)
- Specified by:
submitFailedJob
in interfaceDispatcherGateway
-
listJobs
public CompletableFuture<Collection<JobID>> listJobs(Duration timeout)
Description copied from interface:DispatcherGateway
List the current set of submitted jobs.- Specified by:
listJobs
in interfaceDispatcherGateway
- Parameters:
timeout
- RPC timeout- Returns:
- A future collection of currently submitted jobs
-
disposeSavepoint
public CompletableFuture<Acknowledge> disposeSavepoint(String savepointPath, Duration timeout)
Description copied from interface:RestfulGateway
Dispose the given savepoint.- Specified by:
disposeSavepoint
in interfaceRestfulGateway
- Parameters:
savepointPath
- identifying the savepoint to disposetimeout
- RPC timeout- Returns:
- A future acknowledge if the disposal succeeded
-
cancelJob
public CompletableFuture<Acknowledge> cancelJob(JobID jobId, Duration timeout)
Description copied from interface:RestfulGateway
Cancel the given job.- Specified by:
cancelJob
in interfaceRestfulGateway
- Parameters:
jobId
- identifying the job to canceltimeout
- of the operation- Returns:
- A future acknowledge if the cancellation succeeded
-
requestClusterOverview
public CompletableFuture<ClusterOverview> requestClusterOverview(Duration timeout)
Description copied from interface:RestfulGateway
Requests the cluster status overview.- Specified by:
requestClusterOverview
in interfaceRestfulGateway
- Parameters:
timeout
- for the asynchronous operation- Returns:
- Future containing the status overview
-
requestMultipleJobDetails
public CompletableFuture<MultipleJobsDetails> requestMultipleJobDetails(Duration timeout)
Description copied from interface:RestfulGateway
Requests job details currently being executed on the Flink cluster.- Specified by:
requestMultipleJobDetails
in interfaceRestfulGateway
- Parameters:
timeout
- for the asynchronous operation- Returns:
- Future containing the job details
-
requestJobStatus
public CompletableFuture<JobStatus> requestJobStatus(JobID jobId, Duration timeout)
Description copied from interface:RestfulGateway
Request theJobStatus
of the given job.- Specified by:
requestJobStatus
in interfaceRestfulGateway
- Parameters:
jobId
- identifying the job for which to retrieve the JobStatustimeout
- for the asynchronous operation- Returns:
- A future to the
JobStatus
of the given job
-
requestExecutionGraphInfo
public CompletableFuture<ExecutionGraphInfo> requestExecutionGraphInfo(JobID jobId, Duration timeout)
Description copied from interface:RestfulGateway
Requests theExecutionGraphInfo
containing additional information besides theArchivedExecutionGraph
. If there is no such graph, then the future is completed with aFlinkJobNotFoundException
.- Specified by:
requestExecutionGraphInfo
in interfaceRestfulGateway
- Parameters:
jobId
- identifying the job whoseExecutionGraphInfo
is requestedtimeout
- for the asynchronous operation- Returns:
- Future containing the
ExecutionGraphInfo
for the given jobId, otherwiseFlinkJobNotFoundException
-
requestCheckpointStats
public CompletableFuture<CheckpointStatsSnapshot> requestCheckpointStats(JobID jobId, Duration timeout)
Description copied from interface:RestfulGateway
Requests theCheckpointStatsSnapshot
containing checkpointing information.- Specified by:
requestCheckpointStats
in interfaceRestfulGateway
- Parameters:
jobId
- identifying the job whoseCheckpointStatsSnapshot
is requestedtimeout
- for the asynchronous operation- Returns:
- Future containing the
CheckpointStatsSnapshot
for the given jobId
-
requestJobResult
public CompletableFuture<JobResult> requestJobResult(JobID jobId, Duration timeout)
Description copied from interface:RestfulGateway
Requests theJobResult
of a job specified by the given jobId.- Specified by:
requestJobResult
in interfaceRestfulGateway
- Parameters:
jobId
- identifying the job for which to retrieve theJobResult
.timeout
- for the asynchronous operation- Returns:
- Future which is completed with the job's
JobResult
once the job has finished
-
requestMetricQueryServiceAddresses
public CompletableFuture<Collection<String>> requestMetricQueryServiceAddresses(Duration timeout)
Description copied from interface:RestfulGateway
Requests the addresses of theMetricQueryService
to query.- Specified by:
requestMetricQueryServiceAddresses
in interfaceRestfulGateway
- Parameters:
timeout
- for the asynchronous operation- Returns:
- Future containing the collection of metric query service addresses to query
-
requestTaskManagerMetricQueryServiceAddresses
public CompletableFuture<Collection<Tuple2<ResourceID,String>>> requestTaskManagerMetricQueryServiceAddresses(Duration timeout)
Description copied from interface:RestfulGateway
Requests the addresses for the TaskManagers'MetricQueryService
to query.- Specified by:
requestTaskManagerMetricQueryServiceAddresses
in interfaceRestfulGateway
- Parameters:
timeout
- for the asynchronous operation- Returns:
- Future containing the collection of instance ids and the corresponding metric query service address
-
requestThreadDump
public CompletableFuture<ThreadDumpInfo> requestThreadDump(Duration timeout)
Description copied from interface:RestfulGateway
Requests the thread dump from the JobManager.- Specified by:
requestThreadDump
in interfaceRestfulGateway
- Parameters:
timeout
- timeout of the asynchronous operation- Returns:
- Future containing the thread dump information
-
getBlobServerPort
public CompletableFuture<Integer> getBlobServerPort(Duration timeout)
Description copied from interface:DispatcherGateway
Returns the port of the blob server.- Specified by:
getBlobServerPort
in interfaceDispatcherGateway
- Parameters:
timeout
- of the operation- Returns:
- A future integer of the blob server port
-
triggerCheckpoint
public CompletableFuture<String> triggerCheckpoint(JobID jobID, Duration timeout)
- Specified by:
triggerCheckpoint
in interfaceDispatcherGateway
-
triggerCheckpoint
public CompletableFuture<Acknowledge> triggerCheckpoint(AsynchronousJobOperationKey operationKey, CheckpointType checkpointType, Duration timeout)
Description copied from interface:RestfulGateway
Triggers a checkpoint with the given savepoint directory as a target.- Specified by:
triggerCheckpoint
in interfaceRestfulGateway
- Parameters:
operationKey
- the key of the operation, for deduplication purposescheckpointType
- checkpoint backup type (configured / full / incremental)timeout
- Timeout for the asynchronous operation- Returns:
- A future to the
external pointer
of the savepoint.
-
getTriggeredCheckpointStatus
public CompletableFuture<OperationResult<Long>> getTriggeredCheckpointStatus(AsynchronousJobOperationKey operationKey)
Description copied from interface:RestfulGateway
Get the status of a checkpoint triggered under the specified operation key.- Specified by:
getTriggeredCheckpointStatus
in interfaceRestfulGateway
- Parameters:
operationKey
- key of the operation- Returns:
- Future which completes immediately with the status, or fails if no operation is registered for the key
-
triggerCheckpointAndGetCheckpointID
public CompletableFuture<Long> triggerCheckpointAndGetCheckpointID(JobID jobID, CheckpointType checkpointType, Duration timeout)
Description copied from interface:DispatcherGateway
Triggers a checkpoint, returning a future that completes with the checkpoint id when it is complete.- Specified by:
triggerCheckpointAndGetCheckpointID
in interfaceDispatcherGateway
- Parameters:
jobID
- the job idcheckpointType
- checkpoint type of this checkpoint (configured / full / incremental)timeout
- Timeout for the asynchronous operation- Returns:
- Future which is completed once the operation is triggered successfully
-
triggerSavepoint
public CompletableFuture<Acknowledge> triggerSavepoint(AsynchronousJobOperationKey operationKey, String targetDirectory, SavepointFormatType formatType, TriggerSavepointMode savepointMode, Duration timeout)
Description copied from interface:RestfulGateway
Triggers a savepoint with the given savepoint directory as a target, returning a future that completes when the operation is started.- Specified by:
triggerSavepoint
in interfaceRestfulGateway
- Parameters:
operationKey
- the key of the operation, for deduplication purposestargetDirectory
- Target directory for the savepoint.formatType
- Binary format of the savepoint.savepointMode
- context of the savepoint operationtimeout
- Timeout for the asynchronous operation- Returns:
- Future which is completed once the operation is triggered successfully
-
triggerSavepointAndGetLocation
public CompletableFuture<String> triggerSavepointAndGetLocation(JobID jobId, String targetDirectory, SavepointFormatType formatType, TriggerSavepointMode savepointMode, Duration timeout)
Description copied from interface:DispatcherGateway
Triggers a savepoint with the given savepoint directory as a target, returning a future that completes with the savepoint location when it is complete.- Specified by:
triggerSavepointAndGetLocation
in interfaceDispatcherGateway
- Parameters:
jobId
- the job idtargetDirectory
- Target directory for the savepoint.formatType
- Binary format of the savepoint.savepointMode
- context of the savepoint operationtimeout
- Timeout for the asynchronous operation- Returns:
- Future which is completed once the operation is triggered successfully
-
getTriggeredSavepointStatus
public CompletableFuture<OperationResult<String>> getTriggeredSavepointStatus(AsynchronousJobOperationKey operationKey)
Description copied from interface:RestfulGateway
Get the status of a savepoint triggered under the specified operation key.- Specified by:
getTriggeredSavepointStatus
in interfaceRestfulGateway
- Parameters:
operationKey
- key of the operation- Returns:
- Future which completes immediately with the status, or fails if no operation is registered for the key
-
stopWithSavepoint
public CompletableFuture<Acknowledge> stopWithSavepoint(AsynchronousJobOperationKey operationKey, String targetDirectory, SavepointFormatType formatType, TriggerSavepointMode savepointMode, Duration timeout)
Description copied from interface:RestfulGateway
Stops the job with a savepoint, returning a future that completes when the operation is started.- Specified by:
stopWithSavepoint
in interfaceRestfulGateway
- Parameters:
operationKey
- key of the operation, for deduplicationtargetDirectory
- Target directory for the savepoint.formatType
- Binary format of the savepoint.savepointMode
- context of the savepoint operationtimeout
- for the rpc call- Returns:
- Future which is completed once the operation is triggered successfully
-
stopWithSavepointAndGetLocation
public CompletableFuture<String> stopWithSavepointAndGetLocation(JobID jobId, String targetDirectory, SavepointFormatType formatType, TriggerSavepointMode savepointMode, Duration timeout)
Description copied from interface:DispatcherGateway
Stops the job with a savepoint, returning a future that completes with the savepoint location when the savepoint is completed.- Specified by:
stopWithSavepointAndGetLocation
in interfaceDispatcherGateway
- Parameters:
jobId
- the job idtargetDirectory
- Target directory for the savepoint.savepointMode
- context of the savepoint operationtimeout
- for the rpc call- Returns:
- Future which is completed with the savepoint location once it is completed
-
shutDownCluster
public CompletableFuture<Acknowledge> shutDownCluster()
- Specified by:
shutDownCluster
in interfaceRestfulGateway
-
shutDownCluster
public CompletableFuture<Acknowledge> shutDownCluster(ApplicationStatus applicationStatus)
- Specified by:
shutDownCluster
in interfaceDispatcherGateway
-
deliverCoordinationRequestToCoordinator
public CompletableFuture<CoordinationResponse> deliverCoordinationRequestToCoordinator(JobID jobId, String operatorUid, SerializedValue<CoordinationRequest> serializedRequest, Duration timeout)
Description copied from interface:RestfulGateway
Deliver a coordination request to a specified coordinator and return the response.On the client side, a unique operatorUid must be defined to identify an operator. Otherwise, the query cannot be executed correctly. Note that we use operatorUid instead of operatorID because the latter is an internal runtime concept that cannot be recognized by the client.
- Specified by:
deliverCoordinationRequestToCoordinator
in interfaceRestfulGateway
- Parameters:
jobId
- identifying the job which the coordinator belongs tooperatorUid
- identifying the coordinator to receive the requestserializedRequest
- serialized request to delivertimeout
- RPC timeout- Returns:
- A future containing the response. The response will fail with a
FlinkException
if the task is not running, or no operator/coordinator exists for the given ID, or the coordinator cannot handle client events.
-
reportJobClientHeartbeat
public CompletableFuture<Void> reportJobClientHeartbeat(JobID jobId, long expiredTimestamp, Duration timeout)
Description copied from interface:RestfulGateway
The client reports the heartbeat to the dispatcher for aliveness.- Specified by:
reportJobClientHeartbeat
in interfaceRestfulGateway
-
requestJobResourceRequirements
public CompletableFuture<JobResourceRequirements> requestJobResourceRequirements(JobID jobId)
Description copied from interface:RestfulGateway
Read currentjob resource requirements
for a given job.- Specified by:
requestJobResourceRequirements
in interfaceRestfulGateway
- Parameters:
jobId
- job to read the resource requirements for- Returns:
- Future which that contains current resource requirements.
-
updateJobResourceRequirements
public CompletableFuture<Acknowledge> updateJobResourceRequirements(JobID jobId, JobResourceRequirements jobResourceRequirements)
Description copied from interface:RestfulGateway
Updatejob resource requirements
for a given job. When the returned future is complete the requirements have been updated and were persisted in HA, but the job may not have been rescaled (yet).- Specified by:
updateJobResourceRequirements
in interfaceRestfulGateway
- Parameters:
jobId
- job the given requirements belong tojobResourceRequirements
- new resource requirements for the job- Returns:
- Future which is completed successfully when requirements are updated
-
runPostJobGloballyTerminated
protected void runPostJobGloballyTerminated(JobID jobId, JobStatus jobStatus)
-
onFatalError
protected void onFatalError(Throwable throwable)
-
jobReachedTerminalState
@VisibleForTesting protected CompletableFuture<org.apache.flink.runtime.dispatcher.Dispatcher.CleanupJobState> jobReachedTerminalState(ExecutionGraphInfo executionGraphInfo)
-
onRemovedExecutionPlan
public CompletableFuture<Void> onRemovedExecutionPlan(JobID jobId)
-
-