Interface DispatcherGateway
-
- All Superinterfaces:
FencedRpcGateway<DispatcherId>
,RestfulGateway
,RpcGateway
- All Known Implementing Classes:
Dispatcher
,StandaloneDispatcher
public interface DispatcherGateway extends FencedRpcGateway<DispatcherId>, RestfulGateway
Gateway for the Dispatcher component.
-
-
Method Summary
All Methods Instance Methods Abstract Methods Default Methods Modifier and Type Method Description CompletableFuture<Integer>
getBlobServerPort(Duration timeout)
Returns the port of the blob server.CompletableFuture<Collection<JobID>>
listJobs(Duration timeout)
List the current set of submitted jobs.default CompletableFuture<Acknowledge>
shutDownCluster(ApplicationStatus applicationStatus)
default 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(JobGraph jobGraph, Duration timeout)
Submit a job to the dispatcher.default CompletableFuture<String>
triggerCheckpoint(JobID jobID, Duration timeout)
default 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.default 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.-
Methods inherited from interface org.apache.flink.runtime.rpc.FencedRpcGateway
getFencingToken
-
Methods inherited from interface org.apache.flink.runtime.webmonitor.RestfulGateway
cancelJob, deliverCoordinationRequestToCoordinator, disposeSavepoint, getTriggeredCheckpointStatus, getTriggeredSavepointStatus, reportJobClientHeartbeat, requestCheckpointStats, requestClusterOverview, requestExecutionGraphInfo, requestJob, requestJobResourceRequirements, requestJobResult, requestJobStatus, requestMetricQueryServiceAddresses, requestMultipleJobDetails, requestTaskManagerMetricQueryServiceAddresses, requestThreadDump, shutDownCluster, stopWithSavepoint, triggerCheckpoint, triggerSavepoint, updateJobResourceRequirements
-
Methods inherited from interface org.apache.flink.runtime.rpc.RpcGateway
getAddress, getHostname
-
-
-
-
Method Detail
-
submitJob
CompletableFuture<Acknowledge> submitJob(JobGraph jobGraph, Duration timeout)
Submit a job to the dispatcher.- Parameters:
jobGraph
- JobGraph to submittimeout
- RPC timeout- Returns:
- A future acknowledge if the submission succeeded
-
submitFailedJob
CompletableFuture<Acknowledge> submitFailedJob(JobID jobId, String jobName, Throwable exception)
-
listJobs
CompletableFuture<Collection<JobID>> listJobs(Duration timeout)
List the current set of submitted jobs.- Parameters:
timeout
- RPC timeout- Returns:
- A future collection of currently submitted jobs
-
getBlobServerPort
CompletableFuture<Integer> getBlobServerPort(Duration timeout)
Returns the port of the blob server.- Parameters:
timeout
- of the operation- Returns:
- A future integer of the blob server port
-
shutDownCluster
default CompletableFuture<Acknowledge> shutDownCluster(ApplicationStatus applicationStatus)
-
triggerCheckpoint
default CompletableFuture<String> triggerCheckpoint(JobID jobID, Duration timeout)
-
triggerSavepointAndGetLocation
default 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.- 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
-
stopWithSavepointAndGetLocation
default 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.- 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
-
triggerCheckpointAndGetCheckpointID
default 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.- 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
-
-