Class RawAsyncTableImpl
- All Implemented Interfaces:
AsyncTable<AdvancedScanResultConsumer>
CompletableFuture
will
be finished inside the rpc framework thread, which means that the callbacks registered to the
CompletableFuture
will also be executed inside the rpc framework thread. So users who use
this class should not try to do time consuming tasks in the callbacks.- Since:
- 2.0.0
- See Also:
-
Nested Class Summary
Modifier and TypeClassDescriptionprivate final class
private final class
private static interface
private final class
private static interface
private static interface
Nested classes/interfaces inherited from interface org.apache.hadoop.hbase.client.AsyncTable
AsyncTable.CheckAndMutateBuilder, AsyncTable.CheckAndMutateWithFilterBuilder, AsyncTable.CoprocessorCallback<R>, AsyncTable.CoprocessorServiceBuilder<S,
R> -
Field Summary
Modifier and TypeFieldDescriptionprivate final AsyncConnectionImpl
private final int
private final long
private static final org.slf4j.Logger
private final int
private final long
private final long
private final long
private final long
private final org.apache.hbase.thirdparty.io.netty.util.Timer
private final long
private final long
private final int
private final TableName
private final long
-
Constructor Summary
ConstructorDescriptionRawAsyncTableImpl
(AsyncConnectionImpl conn, org.apache.hbase.thirdparty.io.netty.util.Timer retryTimer, AsyncTableBuilderBase<?> builder) -
Method Summary
Modifier and TypeMethodDescriptionAppends values to one or more columns within a single row.<T> List<CompletableFuture<T>>
Method that does a batch call on Deletes, Gets, Puts, Increments, Appends and RowMutations.private <T> List<CompletableFuture<T>>
private static <REQ,
PREQ, PRESP, RESP>
CompletableFuture<RESP>call
(HBaseRpcController controller, HRegionLocation loc, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.ClientService.Interface stub, REQ req, RawAsyncTableImpl.Converter<PREQ, byte[], REQ> reqConvert, RawAsyncTableImpl.RpcCall<PRESP, PREQ> rpcCall, RawAsyncTableImpl.Converter<RESP, HBaseRpcController, PRESP> respConverter) checkAndMutate
(byte[] row, byte[] family) Atomically checks if a row/family/qualifier value matches the expected value.checkAndMutate
(byte[] row, Filter filter) Atomically checks if a row matches the specified filter.checkAndMutate
(List<CheckAndMutate> checkAndMutates) Batch version of checkAndMutate.checkAndMutate
(CheckAndMutate checkAndMutate) checkAndMutate that atomically checks if a row matches the specified condition.<S,
R> CompletableFuture<R> coprocessorService
(Function<com.google.protobuf.RpcChannel, S> stubMaker, ServiceCaller<S, R> callable, byte[] row) Execute the given coprocessor call on the region which contains the givenrow
.<S,
R> AsyncTable.CoprocessorServiceBuilder<S, R> coprocessorService
(Function<com.google.protobuf.RpcChannel, S> stubMaker, ServiceCaller<S, R> callable, AsyncTable.CoprocessorCallback<R> callback) Execute a coprocessor call on the regions which are covered by a range.private <S,
R> CompletableFuture<R> coprocessorService
(Function<com.google.protobuf.RpcChannel, S> stubMaker, ServiceCaller<S, R> callable, RegionInfo region, byte[] row) Deletes the specified cells/rows in bulk.Deletes the specified cells/row.Extracts certain cells from the given rows, in batch.Extracts certain cells from a given row.private CompletableFuture<Result>
org.apache.hadoop.conf.Configuration
Returns theConfiguration
object used by this instance.Gets theTableDescriptor
for this table.getName()
Gets the fully qualified table name instance of this table.long
getOperationTimeout
(TimeUnit unit) Get timeout of each operation in Table instance.long
getReadRpcTimeout
(TimeUnit unit) Get timeout of each rpc read request in this Table instance.Gets theAsyncTableRegionLocator
for this table.Get the map of request attributeslong
getRpcTimeout
(TimeUnit unit) Get timeout of each rpc request in this Table instance.getScanner
(Scan scan) Returns a scanner on the current table as specified by theScan
object.long
getScanTimeout
(TimeUnit unit) Get the timeout of a single operation in a scan.long
getWriteRpcTimeout
(TimeUnit unit) Get timeout of each rpc write request in this Table instance.Increments one or more columns within a single row.private boolean
locateFinished
(RegionInfo region, byte[] endKey, boolean endKeyInclusive) private static <REQ,
RESP> CompletableFuture<RESP> mutate
(HBaseRpcController controller, HRegionLocation loc, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.ClientService.Interface stub, REQ req, RawAsyncTableImpl.Converter<org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.MutateRequest, byte[], REQ> reqConvert, RawAsyncTableImpl.Converter<RESP, HBaseRpcController, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.MutateResponse> respConverter) mutateRow
(RowMutations mutations) Performs multiple mutations atomically on a single row.private <RES,
RESP> CompletableFuture<RESP> mutateRow
(HBaseRpcController controller, HRegionLocation loc, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.ClientService.Interface stub, RowMutations mutation, RawAsyncTableImpl.Converter<org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.MultiRequest, byte[], RowMutations> reqConvert, Function<RES, RESP> respConverter) private <T> AsyncRpcRetryingCallerFactory.SingleRequestCallerBuilder<T>
newCaller
(byte[] row, int priority, long rpcTimeoutNs) private <T,
R extends OperationWithAttributes & Row>
AsyncRpcRetryingCallerFactory.SingleRequestCallerBuilder<T>newCaller
(R row, long rpcTimeoutNs) private TableOperationSpanBuilder
private <REQ,
RESP> CompletableFuture<RESP> noncedMutate
(long nonceGroup, long nonce, HBaseRpcController controller, HRegionLocation loc, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.ClientService.Interface stub, REQ req, RawAsyncTableImpl.NoncedConverter<org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.MutateRequest, byte[], REQ> reqConvert, RawAsyncTableImpl.Converter<RESP, HBaseRpcController, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.MutateResponse> respConverter) private <S,
R> void onLocateComplete
(Function<com.google.protobuf.RpcChannel, S> stubMaker, ServiceCaller<S, R> callable, AsyncTable.CoprocessorCallback<R> callback, List<HRegionLocation> locs, byte[] endKey, boolean endKeyInclusive, AtomicBoolean locateFinished, AtomicInteger unfinishedRequest, HRegionLocation loc, Throwable error) Puts some data in the table, in batch.Puts some data to the table.private long
resultSize2CacheSize
(long maxResultSize) void
scan
(Scan scan, AdvancedScanResultConsumer consumer) The scan API uses the observer pattern.Return all the results that match the given scan object.private Scan
setDefaultScanConfig
(Scan scan) private static Result
toResult
(HBaseRpcController controller, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.MutateResponse resp) private List<CompletableFuture<Void>>
voidMutate
(List<? extends Row> actions) private static <REQ> CompletableFuture<Void>
voidMutate
(HBaseRpcController controller, HRegionLocation loc, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.ClientService.Interface stub, REQ req, RawAsyncTableImpl.Converter<org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.MutateRequest, byte[], REQ> reqConvert) Methods inherited from class java.lang.Object
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
Methods inherited from interface org.apache.hadoop.hbase.client.AsyncTable
batchAll, checkAndMutateAll, deleteAll, exists, exists, existsAll, getAll, getScanner, getScanner, incrementColumnValue, incrementColumnValue, putAll
-
Field Details
-
LOG
-
conn
-
retryTimer
-
tableName
-
defaultScannerCaching
-
defaultScannerMaxResultSize
-
rpcTimeoutNs
-
readRpcTimeoutNs
-
writeRpcTimeoutNs
-
operationTimeoutNs
-
scanTimeoutNs
-
pauseNs
-
pauseNsForServerOverloaded
-
maxAttempts
-
startLogErrorsCnt
-
requestAttributes
-
-
Constructor Details
-
RawAsyncTableImpl
RawAsyncTableImpl(AsyncConnectionImpl conn, org.apache.hbase.thirdparty.io.netty.util.Timer retryTimer, AsyncTableBuilderBase<?> builder)
-
-
Method Details
-
getName
Description copied from interface:AsyncTable
Gets the fully qualified table name instance of this table.- Specified by:
getName
in interfaceAsyncTable<AdvancedScanResultConsumer>
-
getConfiguration
Description copied from interface:AsyncTable
Returns theConfiguration
object used by this instance.The reference returned is not a copy, so any change made to it will affect this instance.
- Specified by:
getConfiguration
in interfaceAsyncTable<AdvancedScanResultConsumer>
-
getDescriptor
Description copied from interface:AsyncTable
Gets theTableDescriptor
for this table.- Specified by:
getDescriptor
in interfaceAsyncTable<AdvancedScanResultConsumer>
-
getRegionLocator
Description copied from interface:AsyncTable
Gets theAsyncTableRegionLocator
for this table.- Specified by:
getRegionLocator
in interfaceAsyncTable<AdvancedScanResultConsumer>
-
call
private static <REQ,PREQ, CompletableFuture<RESP> callPRESP, RESP> (HBaseRpcController controller, HRegionLocation loc, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.ClientService.Interface stub, REQ req, RawAsyncTableImpl.Converter<PREQ, byte[], REQ> reqConvert, RawAsyncTableImpl.RpcCall<PRESP, PREQ> rpcCall, RawAsyncTableImpl.Converter<RESP, HBaseRpcController, PRESP> respConverter) -
mutate
private static <REQ,RESP> CompletableFuture<RESP> mutate(HBaseRpcController controller, HRegionLocation loc, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.ClientService.Interface stub, REQ req, RawAsyncTableImpl.Converter<org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.MutateRequest, byte[], REQ> reqConvert, RawAsyncTableImpl.Converter<RESP, HBaseRpcController, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.MutateResponse> respConverter) -
voidMutate
private static <REQ> CompletableFuture<Void> voidMutate(HBaseRpcController controller, HRegionLocation loc, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.ClientService.Interface stub, REQ req, RawAsyncTableImpl.Converter<org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.MutateRequest, byte[], REQ> reqConvert) -
toResult
private static Result toResult(HBaseRpcController controller, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.MutateResponse resp) throws IOException - Throws:
IOException
-
noncedMutate
private <REQ,RESP> CompletableFuture<RESP> noncedMutate(long nonceGroup, long nonce, HBaseRpcController controller, HRegionLocation loc, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.ClientService.Interface stub, REQ req, RawAsyncTableImpl.NoncedConverter<org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.MutateRequest, byte[], REQ> reqConvert, RawAsyncTableImpl.Converter<RESP, HBaseRpcController, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.MutateResponse> respConverter) -
newCaller
private <T> AsyncRpcRetryingCallerFactory.SingleRequestCallerBuilder<T> newCaller(byte[] row, int priority, long rpcTimeoutNs) -
newCaller
private <T,R extends OperationWithAttributes & Row> AsyncRpcRetryingCallerFactory.SingleRequestCallerBuilder<T> newCaller(R row, long rpcTimeoutNs) -
get
-
newTableOperationSpanBuilder
-
get
Description copied from interface:AsyncTable
Extracts certain cells from a given row.- Specified by:
get
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
get
- The object that specifies what data to fetch and from which row.- Returns:
- The data coming from the specified row, if it exists. If the row specified doesn't
exist, the
Result
instance returned won't contain anyKeyValue
, as indicated byResult.isEmpty()
. The return value will be wrapped by aCompletableFuture
.
-
put
Description copied from interface:AsyncTable
Puts some data to the table.- Specified by:
put
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
put
- The data to put.- Returns:
- A
CompletableFuture
that always returns null when complete normally.
-
delete
Description copied from interface:AsyncTable
Deletes the specified cells/row.- Specified by:
delete
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
delete
- The object that specifies what to delete.- Returns:
- A
CompletableFuture
that always returns null when complete normally.
-
append
Description copied from interface:AsyncTable
Appends values to one or more columns within a single row.This operation does not appear atomic to readers. Appends are done under a single row lock, so write operations to a row are synchronized, but readers do not take row locks so get and scan operations can see this operation partially completed.
- Specified by:
append
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
append
- object that specifies the columns and amounts to be used for the increment operations- Returns:
- values of columns after the append operation (maybe null). The return value will be
wrapped by a
CompletableFuture
.
-
increment
Description copied from interface:AsyncTable
Increments one or more columns within a single row.This operation does not appear atomic to readers. Increments are done under a single row lock, so write operations to a row are synchronized, but readers do not take row locks so get and scan operations can see this operation partially completed.
- Specified by:
increment
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
increment
- object that specifies the columns and amounts to be used for the increment operations- Returns:
- values of columns after the increment. The return value will be wrapped by a
CompletableFuture
.
-
checkAndMutate
Description copied from interface:AsyncTable
Atomically checks if a row/family/qualifier value matches the expected value. If it does, it adds the Put/Delete/RowMutations.Use the returned
AsyncTable.CheckAndMutateBuilder
to construct your request and then execute it. This is a fluent style API, the code is like:table.checkAndMutate(row, family).qualifier(qualifier).ifNotExists().thenPut(put) .thenAccept(succ -> { if (succ) { System.out.println("Check and put succeeded"); } else { System.out.println("Check and put failed"); } });
- Specified by:
checkAndMutate
in interfaceAsyncTable<AdvancedScanResultConsumer>
-
checkAndMutate
Description copied from interface:AsyncTable
Atomically checks if a row matches the specified filter. If it does, it adds the Put/Delete/RowMutations.Use the returned
AsyncTable.CheckAndMutateWithFilterBuilder
to construct your request and then execute it. This is a fluent style API, the code is like:table.checkAndMutate(row, filter).thenPut(put).thenAccept(succ -> { if (succ) { System.out.println("Check and put succeeded"); } else { System.out.println("Check and put failed"); } });
- Specified by:
checkAndMutate
in interfaceAsyncTable<AdvancedScanResultConsumer>
-
checkAndMutate
Description copied from interface:AsyncTable
checkAndMutate that atomically checks if a row matches the specified condition. If it does, it performs the specified action.- Specified by:
checkAndMutate
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
checkAndMutate
- The CheckAndMutate object.- Returns:
- A
CompletableFuture
s that represent the result for the CheckAndMutate.
-
checkAndMutate
public List<CompletableFuture<CheckAndMutateResult>> checkAndMutate(List<CheckAndMutate> checkAndMutates) Description copied from interface:AsyncTable
Batch version of checkAndMutate. The specified CheckAndMutates are batched only in the sense that they are sent to a RS in one RPC, but each CheckAndMutate operation is still executed atomically (and thus, each may fail independently of others).- Specified by:
checkAndMutate
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
checkAndMutates
- The list of CheckAndMutate.- Returns:
- A list of
CompletableFuture
s that represent the result for each CheckAndMutate.
-
mutateRow
private <RES,RESP> CompletableFuture<RESP> mutateRow(HBaseRpcController controller, HRegionLocation loc, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.ClientService.Interface stub, RowMutations mutation, RawAsyncTableImpl.Converter<org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.MultiRequest, byte[], RowMutations> reqConvert, Function<RES, RESP> respConverter) -
mutateRow
Description copied from interface:AsyncTable
- Specified by:
mutateRow
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
mutations
- object that specifies the set of mutations to perform atomically- Returns:
- A
CompletableFuture
that returns results of Increment/Append operations
-
setDefaultScanConfig
-
scan
Description copied from interface:AsyncTable
The scan API uses the observer pattern.- Specified by:
scan
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
scan
- A configuredScan
object.consumer
- the consumer used to receive results.- See Also:
-
resultSize2CacheSize
-
getScanner
Description copied from interface:AsyncTable
Returns a scanner on the current table as specified by theScan
object.- Specified by:
getScanner
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
scan
- A configuredScan
object.- Returns:
- A scanner.
-
scanAll
Description copied from interface:AsyncTable
Return all the results that match the given scan object.Notice that usually you should use this method with a
Scan
object that has limit set. For example, if you want to get the closest row after a given row, you could do this:table.scanAll(new Scan().withStartRow(row, false).setLimit(1)).thenAccept(results -> { if (results.isEmpty()) { System.out.println("No row after " + Bytes.toStringBinary(row)); } else { System.out.println("The closest row after " + Bytes.toStringBinary(row) + " is " + Bytes.toStringBinary(results.stream().findFirst().get().getRow())); } });
If your result set is very large, you should use other scan method to get a scanner or use callback to process the results. They will do chunking to prevent OOM. The scanAll method will fetch all the results and store them in a List and then return the list to you.
The scan metrics will be collected background if you enable it but you have no way to get it. Usually you can get scan metrics from
ResultScanner
, or throughScanResultConsumer.onScanMetricsCreated
but this method only returns a list of results. So if you really care about scan metrics then you'd better use other scan methods which return aResultScanner
or let you pass in aScanResultConsumer
. There is no performance difference between these scan methods so do not worry.- Specified by:
scanAll
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
scan
- A configuredScan
object. So if you use this method to fetch a really large result set, it is likely to cause OOM.- Returns:
- The results of this small scan operation. The return value will be wrapped by a
CompletableFuture
.
-
get
Description copied from interface:AsyncTable
Extracts certain cells from the given rows, in batch.Notice that you may not get all the results with this function, which means some of the returned
CompletableFuture
s may succeed while some of the other returnedCompletableFuture
s may fail.- Specified by:
get
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
gets
- The objects that specify what data to fetch and from which rows.- Returns:
- A list of
CompletableFuture
s that represent the result for each get.
-
put
Description copied from interface:AsyncTable
Puts some data in the table, in batch.- Specified by:
put
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
puts
- The list of mutations to apply.- Returns:
- A list of
CompletableFuture
s that represent the result for each put.
-
delete
Description copied from interface:AsyncTable
Deletes the specified cells/rows in bulk.- Specified by:
delete
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
deletes
- list of things to delete.- Returns:
- A list of
CompletableFuture
s that represent the result for each delete.
-
batch
Description copied from interface:AsyncTable
Method that does a batch call on Deletes, Gets, Puts, Increments, Appends and RowMutations. The ordering of execution of the actions is not defined. Meaning if you do a Put and a Get in the sameAsyncTable.batch(java.util.List<? extends org.apache.hadoop.hbase.client.Row>)
call, you will not necessarily be guaranteed that the Get returns what the Put had put.- Specified by:
batch
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
actions
- list of Get, Put, Delete, Increment, Append, and RowMutations objects- Returns:
- A list of
CompletableFuture
s that represent the result for each action.
-
voidMutate
-
batch
-
getRpcTimeout
Description copied from interface:AsyncTable
Get timeout of each rpc request in this Table instance. It will be overridden by a more specific rpc timeout config such as readRpcTimeout or writeRpcTimeout.- Specified by:
getRpcTimeout
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
unit
- the unit of time the timeout to be represented in- Returns:
- rpc timeout in the specified time unit
- See Also:
-
getReadRpcTimeout
Description copied from interface:AsyncTable
Get timeout of each rpc read request in this Table instance.- Specified by:
getReadRpcTimeout
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
unit
- the unit of time the timeout to be represented in- Returns:
- read rpc timeout in the specified time unit
-
getWriteRpcTimeout
Description copied from interface:AsyncTable
Get timeout of each rpc write request in this Table instance.- Specified by:
getWriteRpcTimeout
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
unit
- the unit of time the timeout to be represented in- Returns:
- write rpc timeout in the specified time unit
-
getOperationTimeout
Description copied from interface:AsyncTable
Get timeout of each operation in Table instance.- Specified by:
getOperationTimeout
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
unit
- the unit of time the timeout to be represented in- Returns:
- operation rpc timeout in the specified time unit
-
getScanTimeout
Description copied from interface:AsyncTable
Get the timeout of a single operation in a scan. It works like operation timeout for other operations.- Specified by:
getScanTimeout
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
unit
- the unit of time the timeout to be represented in- Returns:
- scan rpc timeout in the specified time unit
-
getRequestAttributes
Description copied from interface:AsyncTable
Get the map of request attributes- Specified by:
getRequestAttributes
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Returns:
- a map of request attributes supplied by the client
-
coprocessorService
private <S,R> CompletableFuture<R> coprocessorService(Function<com.google.protobuf.RpcChannel, S> stubMaker, ServiceCaller<S, R> callable, RegionInfo region, byte[] row) -
coprocessorService
public <S,R> CompletableFuture<R> coprocessorService(Function<com.google.protobuf.RpcChannel, S> stubMaker, ServiceCaller<S, R> callable, byte[] row) Description copied from interface:AsyncTable
Execute the given coprocessor call on the region which contains the givenrow
.The
stubMaker
is just a delegation to thenewStub
call. Usually it is only a one line lambda expression, like:channel -> xxxService.newStub(channel)
- Specified by:
coprocessorService
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Type Parameters:
S
- the type of the asynchronous stubR
- the type of the return value- Parameters:
stubMaker
- a delegation to the actualnewStub
call.callable
- a delegation to the actual protobuf rpc call. See the comment ofServiceCaller
for more details.row
- The row key used to identify the remote region location- Returns:
- the return value of the protobuf rpc call, wrapped by a
CompletableFuture
. - See Also:
-
locateFinished
-
onLocateComplete
private <S,R> void onLocateComplete(Function<com.google.protobuf.RpcChannel, S> stubMaker, ServiceCaller<S, R> callable, AsyncTable.CoprocessorCallback<R> callback, List<HRegionLocation> locs, byte[] endKey, boolean endKeyInclusive, AtomicBoolean locateFinished, AtomicInteger unfinishedRequest, HRegionLocation loc, Throwable error) -
coprocessorService
public <S,R> AsyncTable.CoprocessorServiceBuilder<S,R> coprocessorService(Function<com.google.protobuf.RpcChannel, S> stubMaker, ServiceCaller<S, R> callable, AsyncTable.CoprocessorCallback<R> callback) Description copied from interface:AsyncTable
Execute a coprocessor call on the regions which are covered by a range.Use the returned
AsyncTable.CoprocessorServiceBuilder
construct your request and then execute it.The
stubMaker
is just a delegation to thexxxService.newStub
call. Usually it is only a one line lambda expression, like:channel -> xxxService.newStub(channel)
- Specified by:
coprocessorService
in interfaceAsyncTable<AdvancedScanResultConsumer>
- Parameters:
stubMaker
- a delegation to the actualnewStub
call.callable
- a delegation to the actual protobuf rpc call. See the comment ofServiceCaller
for more details.callback
- callback to get the response. See the comment ofAsyncTable.CoprocessorCallback
for more details.
-