Skip to content

Commit

Permalink
[hotfix] Rename RestfulGateway#requestJobDetails into requestMultiple…
Browse files Browse the repository at this point in the history
…JobDetails to avoid name conflicts with JobMasterGateway
  • Loading branch information
tillrohrmann committed Nov 30, 2017
1 parent b0bd65d commit dcbc966
Show file tree
Hide file tree
Showing 9 changed files with 9 additions and 9 deletions.
Expand Up @@ -223,7 +223,7 @@ public CompletableFuture<Optional<JobManagerMessages.ClassloadingProps>> request
}

@Override
public CompletableFuture<MultipleJobsDetails> requestJobDetails(Time timeout) {
public CompletableFuture<MultipleJobsDetails> requestMultipleJobDetails(Time timeout) {
return FutureUtils.toJava(
jobManagerGateway
.ask(new RequestJobDetails(true, true), FutureUtils.toFiniteDuration(timeout))
Expand Down
Expand Up @@ -330,7 +330,7 @@ public CompletableFuture<ClusterOverview> requestClusterOverview(Time timeout) {
}

@Override
public CompletableFuture<MultipleJobsDetails> requestJobDetails(Time timeout) {
public CompletableFuture<MultipleJobsDetails> requestMultipleJobDetails(Time timeout) {
final int numberJobsRunning = jobManagerRunners.size();

ArrayList<CompletableFuture<JobDetails>> individualJobDetails = new ArrayList<>(numberJobsRunning);
Expand Down
Expand Up @@ -42,8 +42,8 @@
import org.apache.flink.runtime.resourcemanager.ResourceManagerId;
import org.apache.flink.runtime.rpc.FencedRpcGateway;
import org.apache.flink.runtime.rpc.RpcTimeout;
import org.apache.flink.runtime.state.internal.InternalKvState;
import org.apache.flink.runtime.state.KeyGroupRange;
import org.apache.flink.runtime.state.internal.InternalKvState;
import org.apache.flink.runtime.taskexecutor.slot.SlotOffer;
import org.apache.flink.runtime.taskmanager.TaskExecutionState;
import org.apache.flink.runtime.taskmanager.TaskManagerLocation;
Expand Down
Expand Up @@ -59,7 +59,7 @@ protected CompletableFuture<JobIdsWithStatusOverview> handleRequest(
@Nonnull HandlerRequest<EmptyRequestBody, EmptyMessageParameters> request,
@Nonnull RestfulGateway gateway) throws RestHandlerException {

return gateway.requestJobDetails(timeout).thenApply(
return gateway.requestMultipleJobDetails(timeout).thenApply(
multipleJobDetails -> new JobIdsWithStatusOverview(
multipleJobDetails
.getJobs()
Expand Down
Expand Up @@ -55,6 +55,6 @@ public JobsOverviewHandler(

@Override
protected CompletableFuture<MultipleJobsDetails> handleRequest(@Nonnull HandlerRequest<EmptyRequestBody, EmptyMessageParameters> request, @Nonnull RestfulGateway gateway) throws RestHandlerException {
return gateway.requestJobDetails(timeout);
return gateway.requestMultipleJobDetails(timeout);
}
}
Expand Up @@ -68,7 +68,7 @@ public String[] getPaths() {
@Override
public CompletableFuture<String> handleJsonRequest(Map<String, String> pathParams, Map<String, String> queryParams, JobManagerGateway jobManagerGateway) {
if (jobManagerGateway != null) {
CompletableFuture<MultipleJobsDetails> jobDetailsFuture = jobManagerGateway.requestJobDetails(timeout);
CompletableFuture<MultipleJobsDetails> jobDetailsFuture = jobManagerGateway.requestMultipleJobDetails(timeout);

return jobDetailsFuture.thenApplyAsync(
(MultipleJobsDetails result) -> {
Expand Down
Expand Up @@ -106,7 +106,7 @@ private void fetchMetrics() {
/*
* Remove all metrics that belong to a job that is not running and no longer archived.
*/
CompletableFuture<MultipleJobsDetails> jobDetailsFuture = leaderGateway.requestJobDetails(timeout);
CompletableFuture<MultipleJobsDetails> jobDetailsFuture = leaderGateway.requestMultipleJobDetails(timeout);

jobDetailsFuture.whenCompleteAsync(
(MultipleJobsDetails jobDetails, Throwable throwable) -> {
Expand Down
Expand Up @@ -66,7 +66,7 @@ public interface RestfulGateway extends RpcGateway {
* @param timeout for the asynchronous operation
* @return Future containing the job details
*/
CompletableFuture<MultipleJobsDetails> requestJobDetails(
CompletableFuture<MultipleJobsDetails> requestMultipleJobDetails(
@RpcTimeout Time timeout);

/**
Expand Down
Expand Up @@ -81,7 +81,7 @@ public void testUpdate() throws Exception {

JobManagerGateway jobManagerGateway = mock(JobManagerGateway.class);

when(jobManagerGateway.requestJobDetails(any(Time.class)))
when(jobManagerGateway.requestMultipleJobDetails(any(Time.class)))
.thenReturn(CompletableFuture.completedFuture(new MultipleJobsDetails(Collections.emptyList())));
when(jobManagerGateway.requestMetricQueryServicePaths(any(Time.class))).thenReturn(
CompletableFuture.completedFuture(Collections.singleton(jmMetricQueryServicePath)));
Expand Down

0 comments on commit dcbc966

Please sign in to comment.