@InterfaceAudience.Private class RawAsyncTableImpl extends Object implements 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.AsyncTableImpl
Modifier and Type | Class and Description |
---|---|
private class |
RawAsyncTableImpl.CheckAndMutateBuilderImpl |
private static interface |
RawAsyncTableImpl.Converter<D,I,S> |
private class |
RawAsyncTableImpl.CoprocessorServiceBuilderImpl<S,R> |
private static interface |
RawAsyncTableImpl.NoncedConverter<D,I,S> |
private static interface |
RawAsyncTableImpl.RpcCall<RESP,REQ> |
AsyncTable.CheckAndMutateBuilder, AsyncTable.CoprocessorCallback<R>, AsyncTable.CoprocessorServiceBuilder<S,R>
Modifier and Type | Field and Description |
---|---|
private AsyncConnectionImpl |
conn |
private int |
defaultScannerCaching |
private long |
defaultScannerMaxResultSize |
private static org.slf4j.Logger |
LOG |
private int |
maxAttempts |
private long |
operationTimeoutNs |
private long |
pauseForCQTBENs |
private long |
pauseNs |
private long |
readRpcTimeoutNs |
private org.apache.hbase.thirdparty.io.netty.util.Timer |
retryTimer |
private long |
rpcTimeoutNs |
private long |
scanTimeoutNs |
private int |
startLogErrorsCnt |
private TableName |
tableName |
private long |
writeRpcTimeoutNs |
Constructor and Description |
---|
RawAsyncTableImpl(AsyncConnectionImpl conn,
org.apache.hbase.thirdparty.io.netty.util.Timer retryTimer,
AsyncTableBuilderBase<?> builder) |
Modifier and Type | Method and Description |
---|---|
CompletableFuture<Result> |
append(Append append)
Appends values to one or more columns within a single row.
|
<T> List<CompletableFuture<T>> |
batch(List<? extends Row> actions)
Method that does a batch call on Deletes, Gets, Puts, Increments, Appends and RowMutations.
|
private <T> List<CompletableFuture<T>> |
batch(List<? extends Row> actions,
long rpcTimeoutNs) |
private static <REQ,PREQ,PRESP,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) |
AsyncTable.CheckAndMutateBuilder |
checkAndMutate(byte[] row,
byte[] family)
Atomically checks if a row/family/qualifier value matches the expected value.
|
<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.
|
<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 given
row . |
private <S,R> CompletableFuture<R> |
coprocessorService(Function<com.google.protobuf.RpcChannel,S> stubMaker,
ServiceCaller<S,R> callable,
RegionInfo region,
byte[] row) |
CompletableFuture<Void> |
delete(Delete delete)
Deletes the specified cells/row.
|
List<CompletableFuture<Void>> |
delete(List<Delete> deletes)
Deletes the specified cells/rows in bulk.
|
CompletableFuture<Result> |
get(Get get)
Extracts certain cells from a given row.
|
private CompletableFuture<Result> |
get(Get get,
int replicaId) |
List<CompletableFuture<Result>> |
get(List<Get> gets)
Extracts certain cells from the given rows, in batch.
|
org.apache.hadoop.conf.Configuration |
getConfiguration()
Returns the
Configuration object used by this instance. |
TableName |
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.
|
long |
getRpcTimeout(TimeUnit unit)
Get timeout of each rpc request in this Table instance.
|
ResultScanner |
getScanner(Scan scan)
Returns a scanner on the current table as specified by the
Scan 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.
|
CompletableFuture<Result> |
increment(Increment increment)
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) |
private <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<Result,RESP> respConverter) |
CompletableFuture<Void> |
mutateRow(RowMutations mutation)
Performs multiple mutations atomically on a single row.
|
private <T> AsyncRpcRetryingCallerFactory.SingleRequestCallerBuilder<T> |
newCaller(byte[] row,
int priority,
long rpcTimeoutNs) |
private <T,R extends OperationWithAttributes & Row> |
newCaller(R row,
long rpcTimeoutNs) |
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) |
List<CompletableFuture<Void>> |
put(List<Put> puts)
Puts some data in the table, in batch.
|
CompletableFuture<Void> |
put(Put put)
Puts some data to the table.
|
private long |
resultSize2CacheSize(long maxResultSize) |
void |
scan(Scan scan,
AdvancedScanResultConsumer consumer)
The scan API uses the observer pattern.
|
CompletableFuture<List<Result>> |
scanAll(Scan scan)
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 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) |
private List<CompletableFuture<Void>> |
voidMutate(List<? extends Row> actions) |
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
batchAll, deleteAll, exists, exists, existsAll, getAll, getScanner, getScanner, incrementColumnValue, incrementColumnValue, putAll
private static final org.slf4j.Logger LOG
private final AsyncConnectionImpl conn
private final org.apache.hbase.thirdparty.io.netty.util.Timer retryTimer
private final int defaultScannerCaching
private final long defaultScannerMaxResultSize
private final long rpcTimeoutNs
private final long readRpcTimeoutNs
private final long writeRpcTimeoutNs
private final long operationTimeoutNs
private final long scanTimeoutNs
private final long pauseNs
private final long pauseForCQTBENs
private final int maxAttempts
private final int startLogErrorsCnt
RawAsyncTableImpl(AsyncConnectionImpl conn, org.apache.hbase.thirdparty.io.netty.util.Timer retryTimer, AsyncTableBuilderBase<?> builder)
public TableName getName()
AsyncTable
getName
in interface AsyncTable<AdvancedScanResultConsumer>
public org.apache.hadoop.conf.Configuration getConfiguration()
AsyncTable
Configuration
object used by this instance.
The reference returned is not a copy, so any change made to it will affect this instance.
getConfiguration
in interface AsyncTable<AdvancedScanResultConsumer>
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)
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)
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)
private static Result toResult(HBaseRpcController controller, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.MutateResponse resp) throws IOException
IOException
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 <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 CompletableFuture<Result> get(Get get, int replicaId)
public CompletableFuture<Result> get(Get get)
AsyncTable
get
in interface AsyncTable<AdvancedScanResultConsumer>
get
- The object that specifies what data to fetch and from which row.Result
instance returned won't contain any
KeyValue
, as indicated by Result.isEmpty()
. The
return value will be wrapped by a CompletableFuture
.public CompletableFuture<Void> put(Put put)
AsyncTable
put
in interface AsyncTable<AdvancedScanResultConsumer>
put
- The data to put.CompletableFuture
that always returns null when complete normally.public CompletableFuture<Void> delete(Delete delete)
AsyncTable
delete
in interface AsyncTable<AdvancedScanResultConsumer>
delete
- The object that specifies what to delete.CompletableFuture
that always returns null when complete normally.public CompletableFuture<Result> append(Append append)
AsyncTable
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.
append
in interface AsyncTable<AdvancedScanResultConsumer>
append
- object that specifies the columns and amounts to be used for the increment
operationsCompletableFuture
.public CompletableFuture<Result> increment(Increment increment)
AsyncTable
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.
increment
in interface AsyncTable<AdvancedScanResultConsumer>
increment
- object that specifies the columns and amounts to be used for the increment
operationsCompletableFuture
.public AsyncTable.CheckAndMutateBuilder checkAndMutate(byte[] row, byte[] family)
AsyncTable
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");
}
});
checkAndMutate
in interface AsyncTable<AdvancedScanResultConsumer>
private <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<Result,RESP> respConverter)
public CompletableFuture<Void> mutateRow(RowMutations mutation)
AsyncTable
mutateRow
in interface AsyncTable<AdvancedScanResultConsumer>
mutation
- object that specifies the set of mutations to perform atomicallyCompletableFuture
that always returns null when complete normally.private Scan setDefaultScanConfig(Scan scan)
public void scan(Scan scan, AdvancedScanResultConsumer consumer)
AsyncTable
scan
in interface AsyncTable<AdvancedScanResultConsumer>
scan
- A configured Scan
object.consumer
- the consumer used to receive results.ScanResultConsumer
,
AdvancedScanResultConsumer
private long resultSize2CacheSize(long maxResultSize)
public ResultScanner getScanner(Scan scan)
AsyncTable
Scan
object.getScanner
in interface AsyncTable<AdvancedScanResultConsumer>
scan
- A configured Scan
object.public CompletableFuture<List<Result>> scanAll(Scan scan)
AsyncTable
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 through
ScanResultConsumer.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
a ResultScanner
or let you pass in a ScanResultConsumer
. There is no
performance difference between these scan methods so do not worry.
scanAll
in interface AsyncTable<AdvancedScanResultConsumer>
scan
- A configured Scan
object. So if you use this method to fetch a really large
result set, it is likely to cause OOM.CompletableFuture
.public List<CompletableFuture<Result>> get(List<Get> gets)
AsyncTable
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 returned
CompletableFuture
s may fail.
get
in interface AsyncTable<AdvancedScanResultConsumer>
gets
- The objects that specify what data to fetch and from which rows.CompletableFuture
s that represent the result for each get.public List<CompletableFuture<Void>> put(List<Put> puts)
AsyncTable
put
in interface AsyncTable<AdvancedScanResultConsumer>
puts
- The list of mutations to apply.CompletableFuture
s that represent the result for each put.public List<CompletableFuture<Void>> delete(List<Delete> deletes)
AsyncTable
delete
in interface AsyncTable<AdvancedScanResultConsumer>
deletes
- list of things to delete.CompletableFuture
s that represent the result for each delete.public <T> List<CompletableFuture<T>> batch(List<? extends Row> actions)
AsyncTable
AsyncTable.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.batch
in interface AsyncTable<AdvancedScanResultConsumer>
actions
- list of Get, Put, Delete, Increment, Append, and RowMutations objectsCompletableFuture
s that represent the result for each action.private List<CompletableFuture<Void>> voidMutate(List<? extends Row> actions)
private <T> List<CompletableFuture<T>> batch(List<? extends Row> actions, long rpcTimeoutNs)
public long getRpcTimeout(TimeUnit unit)
AsyncTable
getRpcTimeout
in interface AsyncTable<AdvancedScanResultConsumer>
unit
- the unit of time the timeout to be represented inAsyncTable.getReadRpcTimeout(TimeUnit)
,
AsyncTable.getWriteRpcTimeout(TimeUnit)
public long getReadRpcTimeout(TimeUnit unit)
AsyncTable
getReadRpcTimeout
in interface AsyncTable<AdvancedScanResultConsumer>
unit
- the unit of time the timeout to be represented inpublic long getWriteRpcTimeout(TimeUnit unit)
AsyncTable
getWriteRpcTimeout
in interface AsyncTable<AdvancedScanResultConsumer>
unit
- the unit of time the timeout to be represented inpublic long getOperationTimeout(TimeUnit unit)
AsyncTable
getOperationTimeout
in interface AsyncTable<AdvancedScanResultConsumer>
unit
- the unit of time the timeout to be represented inpublic long getScanTimeout(TimeUnit unit)
AsyncTable
getScanTimeout
in interface AsyncTable<AdvancedScanResultConsumer>
unit
- the unit of time the timeout to be represented inprivate <S,R> CompletableFuture<R> coprocessorService(Function<com.google.protobuf.RpcChannel,S> stubMaker, ServiceCaller<S,R> callable, RegionInfo region, byte[] row)
public <S,R> CompletableFuture<R> coprocessorService(Function<com.google.protobuf.RpcChannel,S> stubMaker, ServiceCaller<S,R> callable, byte[] row)
AsyncTable
row
.
The stubMaker
is just a delegation to the newStub
call. Usually it is only a
one line lambda expression, like:
channel -> xxxService.newStub(channel)
coprocessorService
in interface AsyncTable<AdvancedScanResultConsumer>
S
- the type of the asynchronous stubR
- the type of the return valuestubMaker
- a delegation to the actual newStub
call.callable
- a delegation to the actual protobuf rpc call. See the comment of
ServiceCaller
for more details.row
- The row key used to identify the remote region locationCompletableFuture
.ServiceCaller
private boolean locateFinished(RegionInfo region, byte[] endKey, boolean endKeyInclusive)
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)
public <S,R> AsyncTable.CoprocessorServiceBuilder<S,R> coprocessorService(Function<com.google.protobuf.RpcChannel,S> stubMaker, ServiceCaller<S,R> callable, AsyncTable.CoprocessorCallback<R> callback)
AsyncTable
Use the returned AsyncTable.CoprocessorServiceBuilder
construct your request and then execute it.
The stubMaker
is just a delegation to the xxxService.newStub
call. Usually it
is only a one line lambda expression, like:
channel -> xxxService.newStub(channel)
coprocessorService
in interface AsyncTable<AdvancedScanResultConsumer>
stubMaker
- a delegation to the actual newStub
call.callable
- a delegation to the actual protobuf rpc call. See the comment of
ServiceCaller
for more details.callback
- callback to get the response. See the comment of AsyncTable.CoprocessorCallback
for more details.Copyright © 2007–2020 The Apache Software Foundation. All rights reserved.