Package org.apache.flink.client.program
Interface ClusterClient<T>
-
- Type Parameters:
T
- type of the cluster id
- All Superinterfaces:
AutoCloseable
- All Known Implementing Classes:
MiniClusterClient
,RestClusterClient
public interface ClusterClient<T> extends AutoCloseable
Encapsulates the functionality necessary to submit a program to a remote cluster.
-
-
Method Summary
All Methods Instance Methods Abstract Methods Default Methods Modifier and Type Method Description CompletableFuture<Acknowledge>
cancel(JobID jobId)
Cancels a job identified by the job id.CompletableFuture<String>
cancelWithSavepoint(JobID jobId, String savepointDirectory, SavepointFormatType formatType)
Cancels a job identified by the job id and triggers a savepoint.void
close()
CompletableFuture<Acknowledge>
disposeSavepoint(String savepointPath)
Dispose the savepoint under the given path.default CompletableFuture<Map<String,Object>>
getAccumulators(JobID jobID)
Requests and returns the accumulators for the given job identifier.CompletableFuture<Map<String,Object>>
getAccumulators(JobID jobID, ClassLoader loader)
Requests and returns the accumulators for the given job identifier.T
getClusterId()
Returns the cluster id identifying the cluster to which the client is connected.Configuration
getFlinkConfiguration()
Return the Flink configuration object.CompletableFuture<JobStatus>
getJobStatus(JobID jobId)
String
getWebInterfaceURL()
Returns an URL (as a string) to the cluster web interface.default CompletableFuture<Void>
invalidateClusterDataset(AbstractID clusterDatasetId)
Invalidate the cached intermediate dataset with the given id.default CompletableFuture<Set<AbstractID>>
listCompletedClusterDatasetIds()
Return a set of ids of the completed cluster datasets.CompletableFuture<Collection<JobStatusMessage>>
listJobs()
Lists the currently running and finished jobs on the cluster.default CompletableFuture<Void>
reportHeartbeat(JobID jobId, long expiredTimestamp)
The client reports the heartbeat to the dispatcher for aliveness.CompletableFuture<JobResult>
requestJobResult(JobID jobId)
CompletableFuture<CoordinationResponse>
sendCoordinationRequest(JobID jobId, String operatorUid, CoordinationRequest request)
Sends out a request to a specified coordinator and return the response.void
shutDownCluster()
Shut down the cluster that this client communicate with.CompletableFuture<String>
stopWithDetachedSavepoint(JobID jobId, boolean advanceToEndOfEventTime, String savepointDirectory, SavepointFormatType formatType)
Stops a program on Flink cluster whose job-manager is configured in this client's configuration.CompletableFuture<String>
stopWithSavepoint(JobID jobId, boolean advanceToEndOfEventTime, String savepointDirectory, SavepointFormatType formatType)
Stops a program on Flink cluster whose job-manager is configured in this client's configuration.CompletableFuture<JobID>
submitJob(ExecutionPlan executionPlan)
Submit the givenExecutionPlan
to the cluster.CompletableFuture<Long>
triggerCheckpoint(JobID jobId, CheckpointType checkpointType)
Triggers a checkpoint for the job identified by the job id.CompletableFuture<String>
triggerDetachedSavepoint(JobID jobId, String savepointDirectory, SavepointFormatType formatType)
Triggers a detached savepoint for the job identified by the job id.CompletableFuture<String>
triggerSavepoint(JobID jobId, String savepointDirectory, SavepointFormatType formatType)
Triggers a savepoint for the job identified by the job id.
-
-
-
Method Detail
-
close
void close()
- Specified by:
close
in interfaceAutoCloseable
-
getClusterId
T getClusterId()
Returns the cluster id identifying the cluster to which the client is connected.- Returns:
- cluster id of the connected cluster
-
getFlinkConfiguration
Configuration getFlinkConfiguration()
Return the Flink configuration object.- Returns:
- The Flink configuration object
-
shutDownCluster
void shutDownCluster()
Shut down the cluster that this client communicate with.
-
getWebInterfaceURL
String getWebInterfaceURL()
Returns an URL (as a string) to the cluster web interface.
-
listJobs
CompletableFuture<Collection<JobStatusMessage>> listJobs() throws Exception
Lists the currently running and finished jobs on the cluster.- Returns:
- future collection of running and finished jobs
- Throws:
Exception
- if no connection to the cluster could be established
-
disposeSavepoint
CompletableFuture<Acknowledge> disposeSavepoint(String savepointPath) throws FlinkException
Dispose the savepoint under the given path.- Parameters:
savepointPath
- path to the savepoint to be disposed- Returns:
- acknowledge future of the dispose action
- Throws:
FlinkException
-
submitJob
CompletableFuture<JobID> submitJob(ExecutionPlan executionPlan)
Submit the givenExecutionPlan
to the cluster.- Parameters:
executionPlan
- to submit- Returns:
JobID
of the submitted job
-
getJobStatus
CompletableFuture<JobStatus> getJobStatus(JobID jobId)
-
requestJobResult
CompletableFuture<JobResult> requestJobResult(JobID jobId)
-
getAccumulators
default CompletableFuture<Map<String,Object>> getAccumulators(JobID jobID)
Requests and returns the accumulators for the given job identifier. Accumulators can be requested while a is running or after it has finished. The default class loader is used to deserialize the incoming accumulator results.- Parameters:
jobID
- The job identifier of a job.- Returns:
- A Map containing the accumulator's name and its value.
-
getAccumulators
CompletableFuture<Map<String,Object>> getAccumulators(JobID jobID, ClassLoader loader)
Requests and returns the accumulators for the given job identifier. Accumulators can be requested while a is running or after it has finished.- Parameters:
jobID
- The job identifier of a job.loader
- The class loader for deserializing the accumulator results.- Returns:
- A Map containing the accumulator's name and its value.
-
cancel
CompletableFuture<Acknowledge> cancel(JobID jobId)
Cancels a job identified by the job id.- Parameters:
jobId
- the job id
-
cancelWithSavepoint
CompletableFuture<String> cancelWithSavepoint(JobID jobId, @Nullable String savepointDirectory, SavepointFormatType formatType)
Cancels a job identified by the job id and triggers a savepoint.- Parameters:
jobId
- the job idsavepointDirectory
- directory the savepoint should be written toformatType
- a binary format of the savepoint- Returns:
- future of path where the savepoint is located
-
stopWithSavepoint
CompletableFuture<String> stopWithSavepoint(JobID jobId, boolean advanceToEndOfEventTime, @Nullable String savepointDirectory, SavepointFormatType formatType)
Stops a program on Flink cluster whose job-manager is configured in this client's configuration. Stopping works only for streaming programs. Be aware, that the program might continue to run for a while after sending the stop command, because after sources stopped to emit data all operators need to finish processing.- Parameters:
jobId
- the job ID of the streaming program to stopadvanceToEndOfEventTime
- flag indicating if the source should inject aMAX_WATERMARK
in the pipelinesavepointDirectory
- directory the savepoint should be written toformatType
- a binary format of the savepoint- Returns:
- a
CompletableFuture
containing the path where the savepoint is located
-
stopWithDetachedSavepoint
CompletableFuture<String> stopWithDetachedSavepoint(JobID jobId, boolean advanceToEndOfEventTime, @Nullable String savepointDirectory, SavepointFormatType formatType)
Stops a program on Flink cluster whose job-manager is configured in this client's configuration. Stopping works only for streaming programs. Be aware, that the program might continue to run for a while after sending the stop command, because after sources stopped to emit data all operators need to finish processing.- Parameters:
jobId
- the job ID of the streaming program to stopadvanceToEndOfEventTime
- flag indicating if the source should inject aMAX_WATERMARK
in the pipelinesavepointDirectory
- directory the savepoint should be written toformatType
- a binary format of the savepoint- Returns:
- the savepoint trigger id
-
triggerSavepoint
CompletableFuture<String> triggerSavepoint(JobID jobId, @Nullable String savepointDirectory, SavepointFormatType formatType)
Triggers a savepoint for the job identified by the job id. The savepoint will be written to the given savepoint directory, orCheckpointingOptions.SAVEPOINT_DIRECTORY
if it is null.- Parameters:
jobId
- job idsavepointDirectory
- directory the savepoint should be written toformatType
- a binary format of the savepoint- Returns:
- path future where the savepoint is located
-
triggerCheckpoint
CompletableFuture<Long> triggerCheckpoint(JobID jobId, CheckpointType checkpointType)
Triggers a checkpoint for the job identified by the job id. The checkpoint will be written to the checkpoint directory for the job.- Parameters:
jobId
- job idcheckpointType
- the checkpoint type (configured / full / incremental)
-
triggerDetachedSavepoint
CompletableFuture<String> triggerDetachedSavepoint(JobID jobId, @Nullable String savepointDirectory, SavepointFormatType formatType)
Triggers a detached savepoint for the job identified by the job id. The savepoint will be written to the given savepoint directory, orCheckpointingOptions.SAVEPOINT_DIRECTORY
if it is null. Notice that: the detached savepoint will return with a savepoint trigger id instead of the path future, that means the client will return very quickly.- Parameters:
jobId
- job idsavepointDirectory
- directory the savepoint should be written toformatType
- a binary format of the savepoint- Returns:
- the savepoint trigger id
-
sendCoordinationRequest
CompletableFuture<CoordinationResponse> sendCoordinationRequest(JobID jobId, String operatorUid, CoordinationRequest request)
Sends out a 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.
- Parameters:
jobId
- specifies the job which the coordinator belongs tooperatorUid
- specifies which coordinator to receive the requestrequest
- the request to send- Returns:
- the response from the coordinator
-
listCompletedClusterDatasetIds
default CompletableFuture<Set<AbstractID>> listCompletedClusterDatasetIds()
Return a set of ids of the completed cluster datasets.- Returns:
- A set of ids of the completely cached intermediate dataset.
-
invalidateClusterDataset
default CompletableFuture<Void> invalidateClusterDataset(AbstractID clusterDatasetId)
Invalidate the cached intermediate dataset with the given id.- Parameters:
clusterDatasetId
- id of the cluster dataset to be invalidated.- Returns:
- Future which will be completed when the cached dataset is invalidated.
-
reportHeartbeat
default CompletableFuture<Void> reportHeartbeat(JobID jobId, long expiredTimestamp)
The client reports the heartbeat to the dispatcher for aliveness.- Parameters:
jobId
- The jobId for the client and the job.- Returns:
-
-