HBASE-26474 Implement connection-level attributes (addendum)
[hbase.git] / hbase-client / src / main / java / org / apache / hadoop / hbase / client / AsyncServerRequestRpcRetryingCaller.java
blob52a2abe3944079e570e1077736842ff3330f0cc4
1 /**
2 * Licensed to the Apache Software Foundation (ASF) under one
3 * or more contributor license agreements. See the NOTICE file
4 * distributed with this work for additional information
5 * regarding copyright ownership. The ASF licenses this file
6 * to you under the Apache License, Version 2.0 (the
7 * "License"); you may not use this file except in compliance
8 * with the License. You may obtain a copy of the License at
10 * http://www.apache.org/licenses/LICENSE-2.0
12 * Unless required by applicable law or agreed to in writing, software
13 * distributed under the License is distributed on an "AS IS" BASIS,
14 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15 * See the License for the specific language governing permissions and
16 * limitations under the License.
18 package org.apache.hadoop.hbase.client;
20 import static org.apache.hadoop.hbase.util.FutureUtils.addListener;
22 import java.io.IOException;
23 import java.util.concurrent.CompletableFuture;
24 import org.apache.hadoop.hbase.HConstants;
25 import org.apache.hadoop.hbase.ServerName;
26 import org.apache.hadoop.hbase.ipc.HBaseRpcController;
27 import org.apache.yetus.audience.InterfaceAudience;
29 import org.apache.hbase.thirdparty.io.netty.util.Timer;
31 import org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.ClientService;
33 /**
34 * Retry caller for a request call to region server.
35 * Now only used for coprocessor call to region server.
37 @InterfaceAudience.Private
38 public class AsyncServerRequestRpcRetryingCaller<T> extends AsyncRpcRetryingCaller<T> {
40 @FunctionalInterface
41 public interface Callable<T> {
42 CompletableFuture<T> call(HBaseRpcController controller, ClientService.Interface stub);
45 private final Callable<T> callable;
46 private ServerName serverName;
48 public AsyncServerRequestRpcRetryingCaller(Timer retryTimer, AsyncConnectionImpl conn,
49 long pauseNs, long pauseForCQTBENs, int maxAttempts, long operationTimeoutNs,
50 long rpcTimeoutNs, int startLogErrorsCnt, ServerName serverName, Callable<T> callable) {
51 super(retryTimer, conn, HConstants.NORMAL_QOS, pauseNs, pauseForCQTBENs, maxAttempts,
52 operationTimeoutNs, rpcTimeoutNs, startLogErrorsCnt);
53 this.serverName = serverName;
54 this.callable = callable;
57 @Override
58 protected void doCall() {
59 ClientService.Interface stub;
60 try {
61 stub = this.conn.getRegionServerStub(serverName);
62 } catch (IOException e) {
63 onError(e, () -> "Get async admin stub to " + serverName + " failed", err -> {
64 });
65 return;
67 resetCallTimeout();
68 addListener(callable.call(controller, stub), (result, error) -> {
69 if (error != null) {
70 onError(error, () -> "Call to admin stub failed", err -> {
71 });
72 return;
74 future.complete(result);
75 });