HBASE-19213 Align check and mutate operations in Table and AsyncTable

- Deprecates old checkAnd*() operations in Table
- Adds Table#CheckAndMutateBuilder and implements it in HTable

Commiter note: When committing the patch, noticed redundant {@inheritDoc} being added in HTable.
Removed new and olds ones.

Signed-off-by: Apekshit Sharma <appy@apache.org>
This commit is contained in:
Peter Somogyi 2017-12-12 20:29:48 -08:00 committed by Apekshit Sharma
parent 7755a98227
commit 941acc5c05
22 changed files with 527 additions and 315 deletions

View File

@ -240,6 +240,10 @@ public interface AsyncTable<C extends ScanResultConsumerBase> {
*/
CheckAndMutateBuilder ifNotExists();
/**
* Check for equality.
* @param value the expected value
*/
default CheckAndMutateBuilder ifEquals(byte[] value) {
return ifMatches(CompareOperator.EQUAL, value);
}

View File

@ -45,6 +45,7 @@ import org.apache.hadoop.hbase.filter.CompareFilter.CompareOp;
import org.apache.hadoop.hbase.ipc.CoprocessorRpcChannel;
import org.apache.hadoop.hbase.ipc.RpcControllerFactory;
import org.apache.hadoop.hbase.shaded.com.google.common.annotations.VisibleForTesting;
import org.apache.hadoop.hbase.shaded.com.google.common.base.Preconditions;
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
import org.apache.hadoop.hbase.shaded.protobuf.RequestConverter;
import org.apache.hadoop.hbase.shaded.protobuf.ResponseConverter;
@ -206,9 +207,6 @@ public class HTable implements Table {
return conf.getInt(ConnectionConfiguration.MAX_KEYVALUE_SIZE_KEY, -1);
}
/**
* {@inheritDoc}
*/
@Override
public Configuration getConfiguration() {
return configuration;
@ -229,9 +227,6 @@ public class HTable implements Table {
return this.connection;
}
/**
* {@inheritDoc}
*/
@Override
@Deprecated
public HTableDescriptor getTableDescriptor() throws IOException {
@ -362,9 +357,6 @@ public class HTable implements Table {
return getScanner(scan);
}
/**
* {@inheritDoc}
*/
@Override
public Result get(final Get get) throws IOException {
return get(get, get.isCheckExistenceOnly());
@ -405,9 +397,6 @@ public class HTable implements Table {
return callable.call(operationTimeoutMs);
}
/**
* {@inheritDoc}
*/
@Override
public Result[] get(List<Get> gets) throws IOException {
if (gets.size() == 1) {
@ -429,9 +418,6 @@ public class HTable implements Table {
}
}
/**
* {@inheritDoc}
*/
@Override
public void batch(final List<? extends Row> actions, final Object[] results)
throws InterruptedException, IOException {
@ -472,9 +458,6 @@ public class HTable implements Table {
}
}
/**
* {@inheritDoc}
*/
@Override
public <R> void batchCallback(
final List<? extends Row> actions, final Object[] results, final Batch.Callback<R> callback)
@ -505,9 +488,6 @@ public class HTable implements Table {
}
}
/**
* {@inheritDoc}
*/
@Override
public void delete(final Delete delete)
throws IOException {
@ -540,9 +520,6 @@ public class HTable implements Table {
}
}
/**
* {@inheritDoc}
*/
@Override
public void delete(final List<Delete> deletes)
throws IOException {
@ -565,33 +542,24 @@ public class HTable implements Table {
}
}
/**
* {@inheritDoc}
* @throws IOException
*/
@Override
public void put(final Put put) throws IOException {
validatePut(put);
ClientServiceCallable<Void> callable =
new ClientServiceCallable<Void>(this.connection, getName(), put.getRow(),
this.rpcControllerFactory.newController(), put.getPriority()) {
@Override
protected Void rpcCall() throws Exception {
MutateRequest request =
RequestConverter.buildMutateRequest(getLocation().getRegionInfo().getRegionName(),
put);
doMutate(request);
return null;
}
};
@Override
protected Void rpcCall() throws Exception {
MutateRequest request =
RequestConverter.buildMutateRequest(getLocation().getRegionInfo().getRegionName(), put);
doMutate(request);
return null;
}
};
rpcCallerFactory.<Void> newCaller(this.writeRpcTimeoutMs).callWithRetries(callable,
this.operationTimeoutMs);
this.operationTimeoutMs);
}
/**
* {@inheritDoc}
* @throws IOException
*/
@Override
public void put(final List<Put> puts) throws IOException {
for (Put put : puts) {
@ -605,9 +573,6 @@ public class HTable implements Table {
}
}
/**
* {@inheritDoc}
*/
@Override
public void mutateRow(final RowMutations rm) throws IOException {
CancellableRegionServerCallable<MultiResponse> callable =
@ -649,9 +614,6 @@ public class HTable implements Table {
}
}
/**
* {@inheritDoc}
*/
@Override
public Result append(final Append append) throws IOException {
checkHasFamilies(append);
@ -671,9 +633,6 @@ public class HTable implements Table {
callWithRetries(callable, this.operationTimeoutMs);
}
/**
* {@inheritDoc}
*/
@Override
public Result increment(final Increment increment) throws IOException {
checkHasFamilies(increment);
@ -693,9 +652,6 @@ public class HTable implements Table {
this.operationTimeoutMs);
}
/**
* {@inheritDoc}
*/
@Override
public long incrementColumnValue(final byte [] row, final byte [] family,
final byte [] qualifier, final long amount)
@ -703,9 +659,6 @@ public class HTable implements Table {
return incrementColumnValue(row, family, qualifier, amount, Durability.SYNC_WAL);
}
/**
* {@inheritDoc}
*/
@Override
public long incrementColumnValue(final byte [] row, final byte [] family,
final byte [] qualifier, final long amount, final Durability durability)
@ -738,86 +691,81 @@ public class HTable implements Table {
callWithRetries(callable, this.operationTimeoutMs);
}
/**
* {@inheritDoc}
*/
@Override
public boolean checkAndPut(final byte [] row,
final byte [] family, final byte [] qualifier, final byte [] value,
final Put put)
throws IOException {
return checkAndPut(row, family, qualifier, CompareOperator.EQUAL, value, put);
}
private boolean doCheckAndPut(final byte [] row, final byte [] family,
final byte [] qualifier, final String opName,
final byte [] value, final Put put)
throws IOException {
ClientServiceCallable<Boolean> callable =
new ClientServiceCallable<Boolean>(this.connection, getName(), row,
this.rpcControllerFactory.newController(), put.getPriority()) {
@Override
protected Boolean rpcCall() throws Exception {
CompareType compareType = CompareType.valueOf(opName);
MutateRequest request = RequestConverter.buildMutateRequest(
getLocation().getRegionInfo().getRegionName(), row, family, qualifier,
new BinaryComparator(value), compareType, put);
MutateResponse response = doMutate(request);
return Boolean.valueOf(response.getProcessed());
}
};
return rpcCallerFactory.<Boolean> newCaller(this.writeRpcTimeoutMs).
callWithRetries(callable, this.operationTimeoutMs);
}
/**
* {@inheritDoc}
*/
@Override
@Deprecated
public boolean checkAndPut(final byte [] row, final byte [] family,
final byte [] qualifier, final CompareOp compareOp, final byte [] value,
final Put put)
throws IOException {
public boolean checkAndPut(final byte [] row, final byte [] family, final byte [] qualifier,
final byte [] value, final Put put) throws IOException {
return doCheckAndPut(row, family, qualifier, CompareOperator.EQUAL.name(), value, put);
}
@Override
@Deprecated
public boolean checkAndPut(final byte [] row, final byte [] family, final byte [] qualifier,
final CompareOp compareOp, final byte [] value, final Put put) throws IOException {
return doCheckAndPut(row, family, qualifier, compareOp.name(), value, put);
}
/**
* {@inheritDoc}
*/
@Override
public boolean checkAndPut(final byte [] row, final byte [] family,
final byte [] qualifier, final CompareOperator op,
final byte [] value, final Put put)
throws IOException {
@Deprecated
public boolean checkAndPut(final byte [] row, final byte [] family, final byte [] qualifier,
final CompareOperator op, final byte [] value, final Put put) throws IOException {
// The name of the operators in CompareOperator are intentionally those of the
// operators in the filter's CompareOp enum.
return doCheckAndPut(row, family, qualifier, op.name(), value, put);
}
/**
* {@inheritDoc}
*/
@Override
public boolean checkAndDelete(final byte [] row, final byte [] family, final byte [] qualifier,
final byte [] value, final Delete delete) throws IOException {
return checkAndDelete(row, family, qualifier, CompareOperator.EQUAL, value, delete);
private boolean doCheckAndPut(final byte [] row, final byte [] family, final byte [] qualifier,
final String opName, final byte [] value, final Put put) throws IOException {
ClientServiceCallable<Boolean> callable =
new ClientServiceCallable<Boolean>(this.connection, getName(), row,
this.rpcControllerFactory.newController(), put.getPriority()) {
@Override
protected Boolean rpcCall() throws Exception {
CompareType compareType = CompareType.valueOf(opName);
MutateRequest request = RequestConverter.buildMutateRequest(
getLocation().getRegionInfo().getRegionName(), row, family, qualifier,
new BinaryComparator(value), compareType, put);
MutateResponse response = doMutate(request);
return Boolean.valueOf(response.getProcessed());
}
};
return rpcCallerFactory.<Boolean> newCaller(this.writeRpcTimeoutMs)
.callWithRetries(callable, this.operationTimeoutMs);
}
private boolean doCheckAndDelete(final byte [] row, final byte [] family,
final byte [] qualifier, final String opName,
final byte [] value, final Delete delete)
throws IOException {
@Override
@Deprecated
public boolean checkAndDelete(final byte [] row, final byte [] family, final byte [] qualifier,
final byte [] value, final Delete delete) throws IOException {
return doCheckAndDelete(row, family, qualifier, CompareOperator.EQUAL.name(), value, delete);
}
@Override
@Deprecated
public boolean checkAndDelete(final byte [] row, final byte [] family, final byte [] qualifier,
final CompareOp compareOp, final byte [] value, final Delete delete) throws IOException {
return doCheckAndDelete(row, family, qualifier, compareOp.name(), value, delete);
}
@Override
@Deprecated
public boolean checkAndDelete(final byte [] row, final byte [] family, final byte [] qualifier,
final CompareOperator op, final byte [] value, final Delete delete) throws IOException {
return doCheckAndDelete(row, family, qualifier, op.name(), value, delete);
}
private boolean doCheckAndDelete(final byte [] row, final byte [] family, final byte [] qualifier,
final String opName, final byte [] value, final Delete delete) throws IOException {
CancellableRegionServerCallable<SingleResponse> callable =
new CancellableRegionServerCallable<SingleResponse>(
this.connection, getName(), row, this.rpcControllerFactory.newController(), writeRpcTimeoutMs,
new RetryingTimeTracker().start(), delete.getPriority()) {
new CancellableRegionServerCallable<SingleResponse>(
this.connection, getName(), row, this.rpcControllerFactory.newController(),
writeRpcTimeoutMs, new RetryingTimeTracker().start(), delete.getPriority()) {
@Override
protected SingleResponse rpcCall() throws Exception {
CompareType compareType = CompareType.valueOf(opName);
MutateRequest request = RequestConverter.buildMutateRequest(
getLocation().getRegionInfo().getRegionName(), row, family, qualifier,
new BinaryComparator(value), compareType, delete);
getLocation().getRegionInfo().getRegionName(), row, family, qualifier,
new BinaryComparator(value), compareType, delete);
MutateResponse response = doMutate(request);
return ResponseConverter.getResult(request, response, getRpcControllerCellScanner());
}
@ -825,16 +773,16 @@ public class HTable implements Table {
List<Delete> rows = Collections.singletonList(delete);
Object[] results = new Object[1];
AsyncProcessTask task = AsyncProcessTask.newBuilder()
.setPool(pool)
.setTableName(tableName)
.setRowAccess(rows)
.setCallable(callable)
// TODO any better timeout?
.setRpcTimeout(Math.max(readRpcTimeoutMs, writeRpcTimeoutMs))
.setOperationTimeout(operationTimeoutMs)
.setSubmittedRows(AsyncProcessTask.SubmittedRows.ALL)
.setResults(results)
.build();
.setPool(pool)
.setTableName(tableName)
.setRowAccess(rows)
.setCallable(callable)
// TODO any better timeout?
.setRpcTimeout(Math.max(readRpcTimeoutMs, writeRpcTimeoutMs))
.setOperationTimeout(operationTimeoutMs)
.setSubmittedRows(AsyncProcessTask.SubmittedRows.ALL)
.setResults(results)
.build();
AsyncRequestFuture ars = multiAp.submit(task);
ars.waitUntilDone();
if (ars.hasError()) {
@ -843,27 +791,9 @@ public class HTable implements Table {
return ((SingleResponse.Entry)results[0]).isProcessed();
}
/**
* {@inheritDoc}
*/
@Override
@Deprecated
public boolean checkAndDelete(final byte [] row, final byte [] family,
final byte [] qualifier, final CompareOp compareOp, final byte [] value,
final Delete delete)
throws IOException {
return doCheckAndDelete(row, family, qualifier, compareOp.name(), value, delete);
}
/**
* {@inheritDoc}
*/
@Override
public boolean checkAndDelete(final byte [] row, final byte [] family,
final byte [] qualifier, final CompareOperator op,
final byte [] value, final Delete delete)
throws IOException {
return doCheckAndDelete(row, family, qualifier, op.name(), value, delete);
public CheckAndMutateBuilder checkAndMutate(byte[] row, byte[] family) {
return new CheckAndMutateBuilderImpl(row, family);
}
private boolean doCheckAndMutate(final byte [] row, final byte [] family, final byte [] qualifier,
@ -918,9 +848,6 @@ public class HTable implements Table {
return ((Result)results[0]).getExists();
}
/**
* {@inheritDoc}
*/
@Override
@Deprecated
public boolean checkAndMutate(final byte [] row, final byte [] family, final byte [] qualifier,
@ -929,19 +856,13 @@ public class HTable implements Table {
return doCheckAndMutate(row, family, qualifier, compareOp.name(), value, rm);
}
/**
* {@inheritDoc}
*/
@Override
@Deprecated
public boolean checkAndMutate(final byte [] row, final byte [] family, final byte [] qualifier,
final CompareOperator op, final byte [] value, final RowMutations rm)
throws IOException {
final CompareOperator op, final byte [] value, final RowMutations rm) throws IOException {
return doCheckAndMutate(row, family, qualifier, op.name(), value, rm);
}
/**
* {@inheritDoc}
*/
@Override
public boolean exists(final Get get) throws IOException {
Result r = get(get, true);
@ -1054,17 +975,11 @@ public class HTable implements Table {
this.connection.clearRegionCache();
}
/**
* {@inheritDoc}
*/
@Override
public CoprocessorRpcChannel coprocessorService(byte[] row) {
return new RegionCoprocessorRpcChannel(connection, tableName, row);
}
/**
* {@inheritDoc}
*/
@Override
public <T extends Service, R> Map<byte[],R> coprocessorService(final Class<T> service,
byte[] startKey, byte[] endKey, final Batch.Call<T,R> callable)
@ -1082,9 +997,6 @@ public class HTable implements Table {
return results;
}
/**
* {@inheritDoc}
*/
@Override
public <T extends Service, R> void coprocessorService(final Class<T> service,
byte[] startKey, byte[] endKey, final Batch.Call<T,R> callable,
@ -1227,9 +1139,6 @@ public class HTable implements Table {
return results;
}
/**
* {@inheritDoc}
*/
@Override
public <R extends Message> void batchCoprocessorService(
final Descriptors.MethodDescriptor methodDescriptor, final Message request,
@ -1323,4 +1232,62 @@ public class HTable implements Table {
public RegionLocator getRegionLocator() {
return this.locator;
}
private class CheckAndMutateBuilderImpl implements CheckAndMutateBuilder {
private final byte[] row;
private final byte[] family;
private byte[] qualifier;
private CompareOperator op;
private byte[] value;
CheckAndMutateBuilderImpl(byte[] row, byte[] family) {
this.row = Preconditions.checkNotNull(row, "row is null");
this.family = Preconditions.checkNotNull(family, "family is null");
}
@Override
public CheckAndMutateBuilder qualifier(byte[] qualifier) {
this.qualifier = Preconditions.checkNotNull(qualifier, "qualifier is null. Consider using" +
" an empty byte array, or just do not call this method if you want a null qualifier");
return this;
}
@Override
public CheckAndMutateBuilder ifNotExists() {
this.op = CompareOperator.EQUAL;
this.value = null;
return this;
}
@Override
public CheckAndMutateBuilder ifMatches(CompareOperator compareOp, byte[] value) {
this.op = Preconditions.checkNotNull(compareOp, "compareOp is null");
this.value = Preconditions.checkNotNull(value, "value is null");
return this;
}
private void preCheck() {
Preconditions.checkNotNull(op, "condition is null. You need to specify the condition by" +
" calling ifNotExists/ifEquals/ifMatches before executing the request");
}
@Override
public boolean thenPut(Put put) throws IOException {
preCheck();
return doCheckAndPut(row, family, qualifier, op.name(), value, put);
}
@Override
public boolean thenDelete(Delete delete) throws IOException {
preCheck();
return doCheckAndDelete(row, family, qualifier, op.name(), value, delete);
}
@Override
public boolean thenMutate(RowMutations mutation) throws IOException {
preCheck();
return doCheckAndMutate(row, family, qualifier, op.name(), value, mutation);
}
}
}

View File

@ -259,9 +259,11 @@ public interface Table extends Closeable {
* @param put data to put if check succeeds
* @throws IOException e
* @return true if the new put was executed, false otherwise
* @deprecated Since 2.0.0. Will be removed in 3.0.0. Use {@link #checkAndMutate(byte[], byte[])}
*/
@Deprecated
boolean checkAndPut(byte[] row, byte[] family, byte[] qualifier,
byte[] value, Put put) throws IOException;
byte[] value, Put put) throws IOException;
/**
* Atomically checks if a row/family/qualifier value matches the expected
@ -281,12 +283,11 @@ public interface Table extends Closeable {
* @param put data to put if check succeeds
* @throws IOException e
* @return true if the new put was executed, false otherwise
* @deprecated Since 2.0.0. Will be removed in 3.0.0. Use
* {@link #checkAndPut(byte[], byte[], byte[], CompareOperator, byte[], Put)}}
* @deprecated Since 2.0.0. Will be removed in 3.0.0. Use {@link #checkAndMutate(byte[], byte[])}
*/
@Deprecated
boolean checkAndPut(byte[] row, byte[] family, byte[] qualifier,
CompareFilter.CompareOp compareOp, byte[] value, Put put) throws IOException;
CompareFilter.CompareOp compareOp, byte[] value, Put put) throws IOException;
/**
* Atomically checks if a row/family/qualifier value matches the expected
@ -306,9 +307,11 @@ public interface Table extends Closeable {
* @param put data to put if check succeeds
* @throws IOException e
* @return true if the new put was executed, false otherwise
* @deprecated Since 2.0.0. Will be removed in 3.0.0. Use {@link #checkAndMutate(byte[], byte[])}
*/
boolean checkAndPut(byte[] row, byte[] family, byte[] qualifier,
CompareOperator op, byte[] value, Put put) throws IOException;
@Deprecated
boolean checkAndPut(byte[] row, byte[] family, byte[] qualifier, CompareOperator op,
byte[] value, Put put) throws IOException;
/**
* Deletes the specified cells/row.
@ -356,7 +359,9 @@ public interface Table extends Closeable {
* @param delete data to delete if check succeeds
* @throws IOException e
* @return true if the new delete was executed, false otherwise
* @deprecated Since 2.0.0. Will be removed in 3.0.0. Use {@link #checkAndMutate(byte[], byte[])}
*/
@Deprecated
boolean checkAndDelete(byte[] row, byte[] family, byte[] qualifier,
byte[] value, Delete delete) throws IOException;
@ -378,8 +383,7 @@ public interface Table extends Closeable {
* @param delete data to delete if check succeeds
* @throws IOException e
* @return true if the new delete was executed, false otherwise
* @deprecated Since 2.0.0. Will be removed in 3.0.0. Use
* {@link #checkAndDelete(byte[], byte[], byte[], byte[], Delete)}
* @deprecated Since 2.0.0. Will be removed in 3.0.0. Use {@link #checkAndMutate(byte[], byte[])}
*/
@Deprecated
boolean checkAndDelete(byte[] row, byte[] family, byte[] qualifier,
@ -403,10 +407,74 @@ public interface Table extends Closeable {
* @param delete data to delete if check succeeds
* @throws IOException e
* @return true if the new delete was executed, false otherwise
* @deprecated Since 2.0.0. Will be removed in 3.0.0. Use {@link #checkAndMutate(byte[], byte[])}
*/
@Deprecated
boolean checkAndDelete(byte[] row, byte[] family, byte[] qualifier,
CompareOperator op, byte[] value, Delete delete) throws IOException;
/**
* Atomically checks if a row/family/qualifier value matches the expected value. If it does, it
* adds the Put/Delete/RowMutations.
* <p>
* Use the returned {@link CheckAndMutateBuilder} to construct your request and then execute it.
* This is a fluent style API, the code is like:
*
* <pre>
* <code>
* table.checkAndMutate(row, family).qualifier(qualifier).ifNotExists().thenPut(put);
* </code>
* </pre>
*/
CheckAndMutateBuilder checkAndMutate(byte[] row, byte[] family);
/**
* A helper class for sending checkAndMutate request.
*/
interface CheckAndMutateBuilder {
/**
* @param qualifier column qualifier to check.
*/
CheckAndMutateBuilder qualifier(byte[] qualifier);
/**
* Check for lack of column.
*/
CheckAndMutateBuilder ifNotExists();
/**
* Check for equality.
* @param value the expected value
*/
default CheckAndMutateBuilder ifEquals(byte[] value) {
return ifMatches(CompareOperator.EQUAL, value);
}
/**
* @param compareOp comparison operator to use
* @param value the expected value
*/
CheckAndMutateBuilder ifMatches(CompareOperator compareOp, byte[] value);
/**
* @param put data to put if check succeeds
* @return {@code true} if the new put was executed, {@code false} otherwise.
*/
boolean thenPut(Put put) throws IOException;
/**
* @param delete data to delete if check succeeds
* @return {@code true} if the new delete was executed, {@code false} otherwise.
*/
boolean thenDelete(Delete delete) throws IOException;
/**
* @param mutation mutations to perform if check succeeds
* @return true if the new mutation was executed, false otherwise.
*/
boolean thenMutate(RowMutations mutation) throws IOException;
}
/**
* Performs multiple mutations atomically on a single row. Currently
* {@link Put} and {@link Delete} are supported.
@ -649,8 +717,7 @@ public interface Table extends Closeable {
* @param mutation mutations to perform if check succeeds
* @throws IOException e
* @return true if the new put was executed, false otherwise
* @deprecated Since 2.0.0. Will be removed in 3.0.0. Use
* {@link #checkAndMutate(byte[], byte[], byte[], CompareOperator, byte[], RowMutations)}
* @deprecated Since 2.0.0. Will be removed in 3.0.0. Use {@link #checkAndMutate(byte[], byte[])}
*/
@Deprecated
boolean checkAndMutate(byte[] row, byte[] family, byte[] qualifier,
@ -674,7 +741,9 @@ public interface Table extends Closeable {
* @param mutation mutations to perform if check succeeds
* @throws IOException e
* @return true if the new put was executed, false otherwise
* @deprecated Since 2.0.0. Will be removed in 3.0.0. Use {@link #checkAndMutate(byte[], byte[])}
*/
@Deprecated
boolean checkAndMutate(byte[] row, byte[] family, byte[] qualifier, CompareOperator op,
byte[] value, RowMutations mutation) throws IOException;

View File

@ -1892,8 +1892,8 @@ public class PerformanceEvaluation extends Configured implements Tool {
this.table.put(put);
RowMutations mutations = new RowMutations(bytes);
mutations.add(put);
this.table.checkAndMutate(bytes, FAMILY_NAME, getQualifier(), CompareOperator.EQUAL, bytes,
mutations);
this.table.checkAndMutate(bytes, FAMILY_NAME).qualifier(getQualifier())
.ifEquals(bytes).thenMutate(mutations);
}
}
@ -1909,7 +1909,8 @@ public class PerformanceEvaluation extends Configured implements Tool {
Put put = new Put(bytes);
put.addColumn(FAMILY_NAME, getQualifier(), bytes);
this.table.put(put);
this.table.checkAndPut(bytes, FAMILY_NAME, getQualifier(), CompareOperator.EQUAL, bytes, put);
this.table.checkAndMutate(bytes, FAMILY_NAME).qualifier(getQualifier())
.ifEquals(bytes).thenPut(put);
}
}
@ -1927,8 +1928,8 @@ public class PerformanceEvaluation extends Configured implements Tool {
this.table.put(put);
Delete delete = new Delete(put.getRow());
delete.addColumn(FAMILY_NAME, getQualifier());
this.table.checkAndDelete(bytes, FAMILY_NAME, getQualifier(),
CompareOperator.EQUAL, bytes, delete);
this.table.checkAndMutate(bytes, FAMILY_NAME).qualifier(getQualifier())
.ifEquals(bytes).thenDelete(delete);
}
}

View File

@ -363,8 +363,8 @@ public class TableBasedReplicationQueuesImpl extends ReplicationTableBase
*/
private void safeQueueUpdate(RowMutations mutate) throws ReplicationException, IOException{
try (Table replicationTable = getOrBlockOnReplicationTable()) {
boolean updateSuccess = replicationTable.checkAndMutate(mutate.getRow(),
CF_QUEUE, COL_QUEUE_OWNER, CompareOperator.EQUAL, serverNameBytes, mutate);
boolean updateSuccess = replicationTable.checkAndMutate(mutate.getRow(), CF_QUEUE)
.qualifier(COL_QUEUE_OWNER).ifEquals(serverNameBytes).thenMutate(mutate);
if (!updateSuccess) {
throw new ReplicationException("Failed to update Replication Table because we lost queue " +
" ownership");
@ -408,9 +408,9 @@ public class TableBasedReplicationQueuesImpl extends ReplicationTableBase
// server. If it is not then another RS has already claimed it. If it is we set ourselves as the
// new owner and update the queue's history
try (Table replicationTable = getOrBlockOnReplicationTable()) {
boolean success = replicationTable.checkAndMutate(queue.getRow(),
CF_QUEUE, COL_QUEUE_OWNER, CompareOperator.EQUAL, Bytes.toBytes(originalServer),
claimAndRenameQueue);
boolean success = replicationTable.checkAndMutate(queue.getRow(), CF_QUEUE)
.qualifier(COL_QUEUE_OWNER).ifEquals(Bytes.toBytes(originalServer))
.thenMutate(claimAndRenameQueue);
return success;
}
}

View File

@ -509,8 +509,8 @@ public class RowResource extends ResourceBase {
return Response.status(Response.Status.BAD_REQUEST).type(MIMETYPE_TEXT)
.entity("Bad request: The column to put and check do not match." + CRLF).build();
} else {
retValue = table.checkAndPut(key, valueToPutParts[0], valueToPutParts[1],
valueToCheckCell.getValue(), put);
retValue = table.checkAndMutate(key, valueToPutParts[0]).qualifier(valueToPutParts[1])
.ifEquals(valueToCheckCell.getValue()).thenPut(put);
}
} else {
servlet.getMetrics().incrementFailedPutRequests(1);
@ -630,15 +630,15 @@ public class RowResource extends ResourceBase {
if(cellModelCount == 1) {
delete.addColumns(parts[0], parts[1]);
}
retValue = table.checkAndDelete(key, parts[0], parts[1],
valueToDeleteCell.getValue(), delete);
retValue = table.checkAndMutate(key, parts[0]).qualifier(parts[1])
.ifEquals(valueToDeleteCell.getValue()).thenDelete(delete);
} else {
// The case of empty qualifier.
if(cellModelCount == 1) {
delete.addColumns(parts[0], Bytes.toBytes(StringUtils.EMPTY));
}
retValue = table.checkAndDelete(key, parts[0], Bytes.toBytes(StringUtils.EMPTY),
valueToDeleteCell.getValue(), delete);
retValue = table.checkAndMutate(key, parts[0])
.ifEquals(valueToDeleteCell.getValue()).thenDelete(delete);
}
} else {
servlet.getMetrics().incrementFailedDeleteRequests(1);

View File

@ -76,6 +76,8 @@ import java.util.Set;
import java.util.TreeMap;
import java.util.concurrent.TimeUnit;
import org.apache.hadoop.hbase.shaded.com.google.common.base.Preconditions;
/**
* HTable interface to remote tables accessed via REST gateway
*/
@ -666,8 +668,14 @@ public class RemoteHTable implements Table {
}
@Override
@Deprecated
public boolean checkAndPut(byte[] row, byte[] family, byte[] qualifier,
byte[] value, Put put) throws IOException {
return doCheckAndPut(row, family, qualifier, value, put);
}
private boolean doCheckAndPut(byte[] row, byte[] family, byte[] qualifier,
byte[] value, Put put) throws IOException {
// column to check-the-value
put.add(new KeyValue(row, family, qualifier, value));
@ -681,7 +689,7 @@ public class RemoteHTable implements Table {
for (int i = 0; i < maxRetries; i++) {
Response response = client.put(sb.toString(),
Constants.MIMETYPE_PROTOBUF, model.createProtobufOutput());
Constants.MIMETYPE_PROTOBUF, model.createProtobufOutput());
int code = response.getCode();
switch (code) {
case 200:
@ -710,6 +718,7 @@ public class RemoteHTable implements Table {
}
@Override
@Deprecated
public boolean checkAndPut(byte[] row, byte[] family, byte[] qualifier,
CompareOperator compareOp, byte[] value, Put put) throws IOException {
throw new IOException("checkAndPut for non-equal comparison not implemented");
@ -718,6 +727,11 @@ public class RemoteHTable implements Table {
@Override
public boolean checkAndDelete(byte[] row, byte[] family, byte[] qualifier,
byte[] value, Delete delete) throws IOException {
return doCheckAndDelete(row, family, qualifier, value, delete);
}
private boolean doCheckAndDelete(byte[] row, byte[] family, byte[] qualifier,
byte[] value, Delete delete) throws IOException {
Put put = new Put(row);
put.setFamilyCellMap(delete.getFamilyCellMap());
// column to check-the-value
@ -732,7 +746,7 @@ public class RemoteHTable implements Table {
for (int i = 0; i < maxRetries; i++) {
Response response = client.put(sb.toString(),
Constants.MIMETYPE_PROTOBUF, model.createProtobufOutput());
Constants.MIMETYPE_PROTOBUF, model.createProtobufOutput());
int code = response.getCode();
switch (code) {
case 200:
@ -761,11 +775,31 @@ public class RemoteHTable implements Table {
}
@Override
@Deprecated
public boolean checkAndDelete(byte[] row, byte[] family, byte[] qualifier,
CompareOperator compareOp, byte[] value, Delete delete) throws IOException {
throw new IOException("checkAndDelete for non-equal comparison not implemented");
}
@Override
public CheckAndMutateBuilder checkAndMutate(byte[] row, byte[] family) {
return new CheckAndMutateBuilderImpl(row, family);
}
@Override
@Deprecated
public boolean checkAndMutate(byte[] row, byte[] family, byte[] qualifier,
CompareOp compareOp, byte[] value, RowMutations rm) throws IOException {
throw new UnsupportedOperationException("checkAndMutate not implemented");
}
@Override
@Deprecated
public boolean checkAndMutate(byte[] row, byte[] family, byte[] qualifier,
CompareOperator compareOp, byte[] value, RowMutations rm) throws IOException {
throw new UnsupportedOperationException("checkAndMutate not implemented");
}
@Override
public Result increment(Increment increment) throws IOException {
throw new IOException("Increment not supported");
@ -838,19 +872,6 @@ public class RemoteHTable implements Table {
throw new UnsupportedOperationException("batchCoprocessorService not implemented");
}
@Override
@Deprecated
public boolean checkAndMutate(byte[] row, byte[] family, byte[] qualifier,
CompareOp compareOp, byte[] value, RowMutations rm) throws IOException {
throw new UnsupportedOperationException("checkAndMutate not implemented");
}
@Override
public boolean checkAndMutate(byte[] row, byte[] family, byte[] qualifier,
CompareOperator compareOp, byte[] value, RowMutations rm) throws IOException {
throw new UnsupportedOperationException("checkAndMutate not implemented");
}
@Override
@Deprecated
public void setOperationTimeout(int operationTimeout) {
@ -933,4 +954,62 @@ public class RemoteHTable implements Table {
throw new IllegalStateException("URLEncoder doesn't support UTF-8", e);
}
}
private class CheckAndMutateBuilderImpl implements CheckAndMutateBuilder {
private final byte[] row;
private final byte[] family;
private byte[] qualifier;
private byte[] value;
CheckAndMutateBuilderImpl(byte[] row, byte[] family) {
this.row = Preconditions.checkNotNull(row, "row is null");
this.family = Preconditions.checkNotNull(family, "family is null");
}
@Override
public CheckAndMutateBuilder qualifier(byte[] qualifier) {
this.qualifier = Preconditions.checkNotNull(qualifier, "qualifier is null. Consider using" +
" an empty byte array, or just do not call this method if you want a null qualifier");
return this;
}
@Override
public CheckAndMutateBuilder ifNotExists() {
throw new UnsupportedOperationException("CheckAndMutate for non-equal comparison "
+ "not implemented");
}
@Override
public CheckAndMutateBuilder ifMatches(CompareOperator compareOp, byte[] value) {
if (compareOp == CompareOperator.EQUAL) {
this.value = Preconditions.checkNotNull(value, "value is null");
return this;
} else {
throw new UnsupportedOperationException("CheckAndMutate for non-equal comparison " +
"not implemented");
}
}
@Override
public CheckAndMutateBuilder ifEquals(byte[] value) {
this.value = Preconditions.checkNotNull(value, "value is null");
return this;
}
@Override
public boolean thenPut(Put put) throws IOException {
return doCheckAndPut(row, family, qualifier, value, put);
}
@Override
public boolean thenDelete(Delete delete) throws IOException {
return doCheckAndDelete(row, family, qualifier, value, delete);
}
@Override
public boolean thenMutate(RowMutations mutation) throws IOException {
throw new UnsupportedOperationException("thenMutate not implemented");
}
}
}

View File

@ -152,7 +152,8 @@ public class TestRemoteHTableRetries {
public void run() throws Exception {
Put put = new Put(ROW_1);
put.addColumn(COLUMN_1, QUALIFIER_1, VALUE_1);
remoteTable.checkAndPut(ROW_1, COLUMN_1, QUALIFIER_1, VALUE_1, put );
remoteTable.checkAndMutate(ROW_1, COLUMN_1).qualifier(QUALIFIER_1)
.ifEquals(VALUE_1).thenPut(put);
}
});
verify(client, times(RETRIES)).put(anyString(), anyString(), any());
@ -166,7 +167,9 @@ public class TestRemoteHTableRetries {
Put put = new Put(ROW_1);
put.addColumn(COLUMN_1, QUALIFIER_1, VALUE_1);
Delete delete= new Delete(ROW_1);
remoteTable.checkAndDelete(ROW_1, COLUMN_1, QUALIFIER_1, VALUE_1, delete );
//remoteTable.checkAndDelete(ROW_1, COLUMN_1, QUALIFIER_1, VALUE_1, delete );
remoteTable.checkAndMutate(ROW_1, COLUMN_1).qualifier(QUALIFIER_1)
.ifEquals(VALUE_1).thenDelete(delete);
}
});
}

View File

@ -481,17 +481,18 @@ public class TestRemoteTable {
assertEquals(1, remoteTable.existsAll(Collections.singletonList(get)).length);
Delete delete = new Delete(ROW_1);
remoteTable.checkAndDelete(ROW_1, COLUMN_1, QUALIFIER_1, VALUE_1, delete);
remoteTable.checkAndMutate(ROW_1, COLUMN_1).qualifier(QUALIFIER_1)
.ifEquals(VALUE_1).thenDelete(delete);
assertFalse(remoteTable.exists(get));
Put put = new Put(ROW_1);
put.addColumn(COLUMN_1, QUALIFIER_1, VALUE_1);
remoteTable.put(put);
assertTrue(remoteTable.checkAndPut(ROW_1, COLUMN_1, QUALIFIER_1, VALUE_1,
put));
assertFalse(remoteTable.checkAndPut(ROW_1, COLUMN_1, QUALIFIER_1, VALUE_2,
put));
assertTrue(remoteTable.checkAndMutate(ROW_1, COLUMN_1).qualifier(QUALIFIER_1)
.ifEquals(VALUE_1).thenPut(put));
assertFalse(remoteTable.checkAndMutate(ROW_1, COLUMN_1).qualifier(QUALIFIER_1)
.ifEquals(VALUE_2).thenPut(put));
}
/**

View File

@ -18,7 +18,6 @@
*/
package org.apache.hadoop.hbase.client;
import org.apache.hadoop.hbase.CompareOperator;
import org.apache.hadoop.hbase.HBaseTestingUtility;
import org.apache.hadoop.hbase.TableName;
import org.apache.hadoop.hbase.regionserver.NoSuchColumnFamilyException;
@ -124,8 +123,8 @@ public class TestCheckAndMutate {
// put the same row again with C column deleted
RowMutations rm = makeRowMutationsWithColumnCDeleted();
boolean res = table.checkAndMutate(ROWKEY, FAMILY, Bytes.toBytes("A"),
CompareOperator.EQUAL, Bytes.toBytes("a"), rm);
boolean res = table.checkAndMutate(ROWKEY, FAMILY).qualifier(Bytes.toBytes("A"))
.ifEquals(Bytes.toBytes("a")).thenMutate(rm);
assertTrue(res);
// get row back and assert the values
@ -134,8 +133,41 @@ public class TestCheckAndMutate {
//Test that we get a region level exception
try {
rm = getBogusRowMutations();
table.checkAndMutate(ROWKEY, FAMILY, Bytes.toBytes("A"), CompareOperator.EQUAL,
Bytes.toBytes("a"), rm);
table.checkAndMutate(ROWKEY, FAMILY).qualifier(Bytes.toBytes("A"))
.ifEquals(Bytes.toBytes("a")).thenMutate(rm);
fail("Expected NoSuchColumnFamilyException");
} catch (RetriesExhaustedWithDetailsException e) {
try {
throw e.getCause(0);
} catch (NoSuchColumnFamilyException e1) {
// expected
}
}
}
}
@Test
public void testCheckAndMutateWithBuilder() throws Throwable {
try (Table table = createTable()) {
// put one row
putOneRow(table);
// get row back and assert the values
getOneRowAndAssertAllExist(table);
// put the same row again with C column deleted
RowMutations rm = makeRowMutationsWithColumnCDeleted();
boolean res = table.checkAndMutate(ROWKEY, FAMILY).qualifier(Bytes.toBytes("A"))
.ifEquals(Bytes.toBytes("a")).thenMutate(rm);
assertTrue(res);
// get row back and assert the values
getOneRowAndAssertAllButCExist(table);
//Test that we get a region level exception
try {
rm = getBogusRowMutations();
table.checkAndMutate(ROWKEY, FAMILY).qualifier(Bytes.toBytes("A"))
.ifEquals(Bytes.toBytes("a")).thenMutate(rm);
fail("Expected NoSuchColumnFamilyException");
} catch (RetriesExhaustedWithDetailsException e) {
try {

View File

@ -1296,19 +1296,19 @@ public class TestFromClientSide {
table.append(append);
getTestNull(table, ROW, FAMILY, VALUE);
// Work for checkAndMutate, checkAndPut, checkAndDelete
// Work for checkAndMutate using thenPut, thenMutate and thenDelete
put = new Put(ROW);
put.addColumn(FAMILY, null, Bytes.toBytes("checkAndPut"));
table.put(put);
table.checkAndPut(ROW, FAMILY, null, VALUE, put);
table.checkAndMutate(ROW, FAMILY).ifEquals(VALUE).thenPut(put);
RowMutations mutate = new RowMutations(ROW);
mutate.add(new Put(ROW).addColumn(FAMILY, null, Bytes.toBytes("checkAndMutate")));
table.checkAndMutate(ROW, FAMILY, null, CompareOperator.EQUAL, Bytes.toBytes("checkAndPut"), mutate);
table.checkAndMutate(ROW, FAMILY).ifEquals(Bytes.toBytes("checkAndPut")).thenMutate(mutate);
delete = new Delete(ROW);
delete.addColumns(FAMILY, null);
table.checkAndDelete(ROW, FAMILY, null, Bytes.toBytes("checkAndMutate"), delete);
table.checkAndMutate(ROW, FAMILY).ifEquals(Bytes.toBytes("checkAndMutate")).thenDelete(delete);
}
@Test
@ -4790,22 +4790,23 @@ public class TestFromClientSide {
put1.addColumn(FAMILY, QUALIFIER, VALUE);
// row doesn't exist, so using non-null value should be considered "not match".
boolean ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, VALUE, put1);
boolean ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifEquals(VALUE).thenPut(put1);
assertEquals(ok, false);
// row doesn't exist, so using "null" to check for existence should be considered "match".
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, null, put1);
// row doesn't exist, so using "ifNotExists" should be considered "match".
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER).ifNotExists().thenPut(put1);
assertEquals(ok, true);
// row now exists, so using "null" to check for existence should be considered "not match".
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, null, put1);
// row now exists, so using "ifNotExists" should be considered "not match".
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER).ifNotExists().thenPut(put1);
assertEquals(ok, false);
Put put2 = new Put(ROW);
put2.addColumn(FAMILY, QUALIFIER, value2);
// row now exists, use the matching value to check
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, VALUE, put2);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER).ifEquals(VALUE).thenPut(put2);
assertEquals(ok, true);
Put put3 = new Put(anotherrow);
@ -4813,8 +4814,8 @@ public class TestFromClientSide {
// try to do CheckAndPut on different rows
try {
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, value2, put3);
fail("trying to check and modify different rows should have failed.");
table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER).ifEquals(value2).thenPut(put3);
fail("trying to check and modify different rows should have failed.");
} catch(Exception e) {}
}
@ -4834,53 +4835,71 @@ public class TestFromClientSide {
Put put3 = new Put(ROW);
put3.addColumn(FAMILY, QUALIFIER, value3);
// row doesn't exist, so using "null" to check for existence should be considered "match".
boolean ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, null, put2);
// row doesn't exist, so using "ifNotExists" should be considered "match".
boolean ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER).ifNotExists().thenPut(put2);
assertEquals(ok, true);
// cell = "bbbb", using "aaaa" to compare only LESS/LESS_OR_EQUAL/NOT_EQUAL
// turns out "match"
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, CompareOperator.GREATER, value1, put2);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.GREATER, value1).thenPut(put2);
assertEquals(ok, false);
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, CompareOperator.EQUAL, value1, put2);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.EQUAL, value1).thenPut(put2);
assertEquals(ok, false);
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, CompareOperator.GREATER_OR_EQUAL, value1, put2);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.GREATER_OR_EQUAL, value1).thenPut(put2);
assertEquals(ok, false);
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, CompareOperator.LESS, value1, put2);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.LESS, value1).thenPut(put2);
assertEquals(ok, true);
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, CompareOperator.LESS_OR_EQUAL, value1, put2);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.LESS_OR_EQUAL, value1).thenPut(put2);
assertEquals(ok, true);
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, CompareOperator.NOT_EQUAL, value1, put3);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.NOT_EQUAL, value1).thenPut(put3);
assertEquals(ok, true);
// cell = "cccc", using "dddd" to compare only LARGER/LARGER_OR_EQUAL/NOT_EQUAL
// turns out "match"
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, CompareOperator.LESS, value4, put3);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.LESS, value4).thenPut(put3);
assertEquals(ok, false);
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, CompareOperator.LESS_OR_EQUAL, value4, put3);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.LESS_OR_EQUAL, value4).thenPut(put3);
assertEquals(ok, false);
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, CompareOperator.EQUAL, value4, put3);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.EQUAL, value4).thenPut(put3);
assertEquals(ok, false);
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, CompareOperator.GREATER, value4, put3);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.GREATER, value4).thenPut(put3);
assertEquals(ok, true);
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, CompareOperator.GREATER_OR_EQUAL, value4, put3);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.GREATER_OR_EQUAL, value4).thenPut(put3);
assertEquals(ok, true);
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, CompareOperator.NOT_EQUAL, value4, put2);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.NOT_EQUAL, value4).thenPut(put2);
assertEquals(ok, true);
// cell = "bbbb", using "bbbb" to compare only GREATER_OR_EQUAL/LESS_OR_EQUAL/EQUAL
// turns out "match"
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, CompareOperator.GREATER, value2, put2);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.GREATER, value2).thenPut(put2);
assertEquals(ok, false);
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, CompareOperator.NOT_EQUAL, value2, put2);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.NOT_EQUAL, value2).thenPut(put2);
assertEquals(ok, false);
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, CompareOperator.LESS, value2, put2);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.LESS, value2).thenPut(put2);
assertEquals(ok, false);
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, CompareOperator.GREATER_OR_EQUAL, value2, put2);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.GREATER_OR_EQUAL, value2).thenPut(put2);
assertEquals(ok, true);
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, CompareOperator.LESS_OR_EQUAL, value2, put2);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.LESS_OR_EQUAL, value2).thenPut(put2);
assertEquals(ok, true);
ok = table.checkAndPut(ROW, FAMILY, QUALIFIER, CompareOperator.EQUAL, value2, put3);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.EQUAL, value2).thenPut(put3);
assertEquals(ok, true);
}
@ -4898,7 +4917,8 @@ public class TestFromClientSide {
Delete delete = new Delete(ROW);
delete.addColumns(FAMILY, QUALIFIER);
boolean ok = table.checkAndDelete(ROW, FAMILY, QUALIFIER, value1, delete);
boolean ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifEquals(value1).thenDelete(delete);
assertEquals(ok, true);
}
@ -4924,55 +4944,73 @@ public class TestFromClientSide {
// cell = "bbbb", using "aaaa" to compare only LESS/LESS_OR_EQUAL/NOT_EQUAL
// turns out "match"
boolean ok = table.checkAndDelete(ROW, FAMILY, QUALIFIER, CompareOperator.GREATER, value1, delete);
boolean ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.GREATER, value1).thenDelete(delete);
assertEquals(ok, false);
ok = table.checkAndDelete(ROW, FAMILY, QUALIFIER, CompareOperator.EQUAL, value1, delete);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.EQUAL, value1).thenDelete(delete);
assertEquals(ok, false);
ok = table.checkAndDelete(ROW, FAMILY, QUALIFIER, CompareOperator.GREATER_OR_EQUAL, value1, delete);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.GREATER_OR_EQUAL, value1).thenDelete(delete);
assertEquals(ok, false);
ok = table.checkAndDelete(ROW, FAMILY, QUALIFIER, CompareOperator.LESS, value1, delete);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.LESS, value1).thenDelete(delete);
assertEquals(ok, true);
table.put(put2);
ok = table.checkAndDelete(ROW, FAMILY, QUALIFIER, CompareOperator.LESS_OR_EQUAL, value1, delete);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.LESS_OR_EQUAL, value1).thenDelete(delete);
assertEquals(ok, true);
table.put(put2);
ok = table.checkAndDelete(ROW, FAMILY, QUALIFIER, CompareOperator.NOT_EQUAL, value1, delete);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.NOT_EQUAL, value1).thenDelete(delete);
assertEquals(ok, true);
// cell = "cccc", using "dddd" to compare only LARGER/LARGER_OR_EQUAL/NOT_EQUAL
// turns out "match"
table.put(put3);
ok = table.checkAndDelete(ROW, FAMILY, QUALIFIER, CompareOperator.LESS, value4, delete);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.LESS, value4).thenDelete(delete);
assertEquals(ok, false);
ok = table.checkAndDelete(ROW, FAMILY, QUALIFIER, CompareOperator.LESS_OR_EQUAL, value4, delete);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.LESS_OR_EQUAL, value4).thenDelete(delete);
assertEquals(ok, false);
ok = table.checkAndDelete(ROW, FAMILY, QUALIFIER, CompareOperator.EQUAL, value4, delete);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.EQUAL, value4).thenDelete(delete);
assertEquals(ok, false);
ok = table.checkAndDelete(ROW, FAMILY, QUALIFIER, CompareOperator.GREATER, value4, delete);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.GREATER, value4).thenDelete(delete);
assertEquals(ok, true);
table.put(put3);
ok = table.checkAndDelete(ROW, FAMILY, QUALIFIER, CompareOperator.GREATER_OR_EQUAL, value4, delete);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.GREATER_OR_EQUAL, value4).thenDelete(delete);
assertEquals(ok, true);
table.put(put3);
ok = table.checkAndDelete(ROW, FAMILY, QUALIFIER, CompareOperator.NOT_EQUAL, value4, delete);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.NOT_EQUAL, value4).thenDelete(delete);
assertEquals(ok, true);
// cell = "bbbb", using "bbbb" to compare only GREATER_OR_EQUAL/LESS_OR_EQUAL/EQUAL
// turns out "match"
table.put(put2);
ok = table.checkAndDelete(ROW, FAMILY, QUALIFIER, CompareOperator.GREATER, value2, delete);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.GREATER, value2).thenDelete(delete);
assertEquals(ok, false);
ok = table.checkAndDelete(ROW, FAMILY, QUALIFIER, CompareOperator.NOT_EQUAL, value2, delete);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.NOT_EQUAL, value2).thenDelete(delete);
assertEquals(ok, false);
ok = table.checkAndDelete(ROW, FAMILY, QUALIFIER, CompareOperator.LESS, value2, delete);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.LESS, value2).thenDelete(delete);
assertEquals(ok, false);
ok = table.checkAndDelete(ROW, FAMILY, QUALIFIER, CompareOperator.GREATER_OR_EQUAL, value2, delete);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.GREATER_OR_EQUAL, value2).thenDelete(delete);
assertEquals(ok, true);
table.put(put2);
ok = table.checkAndDelete(ROW, FAMILY, QUALIFIER, CompareOperator.LESS_OR_EQUAL, value2, delete);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.LESS_OR_EQUAL, value2).thenDelete(delete);
assertEquals(ok, true);
table.put(put2);
ok = table.checkAndDelete(ROW, FAMILY, QUALIFIER, CompareOperator.EQUAL, value2, delete);
ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
.ifMatches(CompareOperator.EQUAL, value2).thenDelete(delete);
assertEquals(ok, true);
}

View File

@ -256,7 +256,7 @@ public class TestRegionObserverInterface {
verifyMethodResult(SimpleRegionObserver.class,
new String[] { "hadPreCheckAndPut", "hadPreCheckAndPutAfterRowLock", "hadPostCheckAndPut" },
tableName, new Boolean[] { false, false, false });
table.checkAndPut(Bytes.toBytes(0), A, A, A, p);
table.checkAndMutate(Bytes.toBytes(0), A).qualifier(A).ifEquals(A).thenPut(p);
verifyMethodResult(SimpleRegionObserver.class,
new String[] { "hadPreCheckAndPut", "hadPreCheckAndPutAfterRowLock", "hadPostCheckAndPut" },
tableName, new Boolean[] { true, true, true });
@ -279,7 +279,7 @@ public class TestRegionObserverInterface {
SimpleRegionObserver.class, new String[] { "hadPreCheckAndDelete",
"hadPreCheckAndDeleteAfterRowLock", "hadPostCheckAndDelete" },
tableName, new Boolean[] { false, false, false });
table.checkAndDelete(Bytes.toBytes(0), A, A, A, d);
table.checkAndMutate(Bytes.toBytes(0), A).qualifier(A).ifEquals(A).thenDelete(d);
verifyMethodResult(
SimpleRegionObserver.class, new String[] { "hadPreCheckAndDelete",
"hadPreCheckAndDeleteAfterRowLock", "hadPostCheckAndDelete" },

View File

@ -214,6 +214,7 @@ public class RegionAsTable implements Table {
}
@Override
@Deprecated
public boolean checkAndPut(byte[] row, byte[] family, byte[] qualifier, byte[] value, Put put)
throws IOException {
throw new UnsupportedOperationException();
@ -228,6 +229,7 @@ public class RegionAsTable implements Table {
}
@Override
@Deprecated
public boolean checkAndPut(byte[] row, byte[] family, byte[] qualifier,
CompareOperator compareOp, byte[] value, Put put)
throws IOException {
@ -260,12 +262,18 @@ public class RegionAsTable implements Table {
}
@Override
@Deprecated
public boolean checkAndDelete(byte[] row, byte[] family, byte[] qualifier,
CompareOperator compareOp, byte[] value, Delete delete)
throws IOException {
throw new UnsupportedOperationException();
}
@Override
public CheckAndMutateBuilder checkAndMutate(byte[] row, byte[] family) {
throw new UnsupportedOperationException();
}
@Override
public void mutateRow(RowMutations rm) throws IOException {
throw new UnsupportedOperationException();
@ -344,10 +352,9 @@ public class RegionAsTable implements Table {
}
@Override
@Deprecated
public boolean checkAndMutate(byte[] row, byte[] family, byte[] qualifier,
CompareOperator compareOp,
byte[] value, RowMutations mutation)
throws IOException {
CompareOperator compareOp, byte[] value, RowMutations mutation) throws IOException {
throw new UnsupportedOperationException();
}

View File

@ -382,11 +382,11 @@ public class TestRegionServerMetrics {
Put pTwo = new Put(row);
pTwo.addColumn(cf, qualifier, valTwo);
table.checkAndPut(row, cf, qualifier, valOne, pTwo);
table.checkAndMutate(row, cf).qualifier(qualifier).ifEquals(valOne).thenPut(pTwo);
Put pThree = new Put(row);
pThree.addColumn(cf, qualifier, valThree);
table.checkAndPut(row, cf, qualifier, valOne, pThree);
table.checkAndMutate(row, cf).qualifier(qualifier).ifEquals(valOne).thenPut(pThree);
metricsRegionServer.getRegionServerWrapper().forceRecompute();
assertCounter("checkMutateFailedCount", 1);

View File

@ -22,7 +22,6 @@ import org.apache.commons.logging.Log;
import org.apache.commons.logging.LogFactory;
import org.apache.hadoop.hbase.Cell;
import org.apache.hadoop.hbase.ClusterStatus.Option;
import org.apache.hadoop.hbase.CompareOperator;
import org.apache.hadoop.hbase.HBaseTestingUtility;
import org.apache.hadoop.hbase.RegionLoad;
import org.apache.hadoop.hbase.ServerLoad;
@ -301,7 +300,7 @@ public class TestRegionServerReadRequestMetrics {
put = new Put(ROW1);
put.addColumn(CF1, COL2, VAL2);
boolean checkAndPut =
table.checkAndPut(ROW1, CF1, COL2, CompareOperator.EQUAL, VAL2, put);
table.checkAndMutate(ROW1, CF1).qualifier(COL2).ifEquals(VAL2).thenPut(put);
resultCount = checkAndPut ? 1 : 0;
testReadRequests(resultCount, 1, 0);
@ -318,7 +317,7 @@ public class TestRegionServerReadRequestMetrics {
RowMutations rm = new RowMutations(ROW1);
rm.add(put);
boolean checkAndMutate =
table.checkAndMutate(ROW1, CF1, COL1, CompareOperator.EQUAL, VAL1, rm);
table.checkAndMutate(ROW1, CF1).qualifier(COL1).ifEquals(VAL1).thenMutate(rm);
resultCount = checkAndMutate ? 1 : 0;
testReadRequests(resultCount, 1, 0);
}

View File

@ -934,8 +934,8 @@ public class TestAccessController extends SecureTestUtil {
d.addFamily(TEST_FAMILY);
try(Connection conn = ConnectionFactory.createConnection(conf);
Table t = conn.getTable(TEST_TABLE);) {
t.checkAndDelete(TEST_ROW, TEST_FAMILY, TEST_QUALIFIER,
Bytes.toBytes("test_value"), d);
t.checkAndMutate(TEST_ROW, TEST_FAMILY).qualifier(TEST_QUALIFIER)
.ifEquals(Bytes.toBytes("test_value")).thenDelete(d);
}
return null;
}
@ -949,9 +949,9 @@ public class TestAccessController extends SecureTestUtil {
Put p = new Put(TEST_ROW);
p.addColumn(TEST_FAMILY, TEST_QUALIFIER, Bytes.toBytes(1));
try(Connection conn = ConnectionFactory.createConnection(conf);
Table t = conn.getTable(TEST_TABLE);) {
t.checkAndPut(TEST_ROW, TEST_FAMILY, TEST_QUALIFIER,
Bytes.toBytes("test_value"), p);
Table t = conn.getTable(TEST_TABLE)) {
t.checkAndMutate(TEST_ROW, TEST_FAMILY).qualifier(TEST_QUALIFIER)
.ifEquals(Bytes.toBytes("test_value")).thenPut(p);
}
return null;
}

View File

@ -905,7 +905,8 @@ public class TestCellACLWithMultipleVersions extends SecureTestUtil {
try (Table t = connection.getTable(TEST_TABLE.getTableName())) {
Delete d = new Delete(TEST_ROW1);
d.addColumns(TEST_FAMILY1, TEST_Q1, 120);
t.checkAndDelete(TEST_ROW1, TEST_FAMILY1, TEST_Q1, ZERO, d);
t.checkAndMutate(TEST_ROW1, TEST_FAMILY1).qualifier(TEST_Q1)
.ifEquals(ZERO).thenDelete(d);
}
}
return null;
@ -941,7 +942,7 @@ public class TestCellACLWithMultipleVersions extends SecureTestUtil {
try (Table t = connection.getTable(TEST_TABLE.getTableName())) {
Delete d = new Delete(row);
d.addColumn(TEST_FAMILY1, q1, 120);
t.checkAndDelete(row, TEST_FAMILY1, q1, value, d);
t.checkAndMutate(row, TEST_FAMILY1).qualifier(q1).ifEquals(value).thenDelete(d);
}
}
return null;
@ -958,7 +959,7 @@ public class TestCellACLWithMultipleVersions extends SecureTestUtil {
try (Table t = connection.getTable(TEST_TABLE.getTableName())) {
Delete d = new Delete(row);
d.addColumns(TEST_FAMILY1, TEST_Q1);
t.checkAndDelete(row, TEST_FAMILY1, TEST_Q1, value, d);
t.checkAndMutate(row, TEST_FAMILY1).qualifier(TEST_Q1).ifEquals(value).thenDelete(d);
fail(user.getShortName() + " should not be allowed to do checkAndDelete");
} catch (Exception e) {
}

View File

@ -569,12 +569,12 @@ public abstract class TestVisibilityLabels {
Put put = new Put(row1);
put.addColumn(fam, qual, HConstants.LATEST_TIMESTAMP, value);
put.setCellVisibility(new CellVisibility(SECRET + " & " + CONFIDENTIAL));
table.checkAndPut(row1, fam, qual, null, put);
table.checkAndMutate(row1, fam).qualifier(qual).ifNotExists().thenPut(put);
byte[] row2 = Bytes.toBytes("row2");
put = new Put(row2);
put.addColumn(fam, qual, HConstants.LATEST_TIMESTAMP, value);
put.setCellVisibility(new CellVisibility(SECRET));
table.checkAndPut(row2, fam, qual, null, put);
table.checkAndMutate(row2, fam).qualifier(qual).ifNotExists().thenPut(put);
Scan scan = new Scan();
scan.setAuthorizations(new Authorizations(SECRET));

View File

@ -287,9 +287,9 @@ public class MultiThreadedUpdater extends MultiThreadedWriterBase {
} else if (m instanceof Append) {
table.append((Append)m);
} else if (m instanceof Put) {
table.checkAndPut(row, cf, q, v, (Put)m);
table.checkAndMutate(row, cf).qualifier(q).ifEquals(v).thenPut((Put)m);
} else if (m instanceof Delete) {
table.checkAndDelete(row, cf, q, v, (Delete)m);
table.checkAndMutate(row, cf).qualifier(q).ifEquals(v).thenDelete((Delete)m);
} else {
throw new IllegalArgumentException(
"unsupported mutation " + m.getClass().getSimpleName());
@ -340,9 +340,9 @@ public class MultiThreadedUpdater extends MultiThreadedWriterBase {
} else if (m instanceof Append) {
table.append((Append)m);
} else if (m instanceof Put) {
table.checkAndPut(row, cf, q, v, (Put)m);
table.checkAndMutate(row, cf).qualifier(q).ifEquals(v).thenPut((Put)m);
} else if (m instanceof Delete) {
table.checkAndDelete(row, cf, q, v, (Delete)m);
table.checkAndMutate(row, cf).qualifier(q).ifEquals(v).thenDelete((Delete)m);
} else {
throw new IllegalArgumentException(
"unsupported mutation " + m.getClass().getSimpleName());

View File

@ -233,9 +233,9 @@ public class MultiThreadedUpdaterWithACL extends MultiThreadedUpdater {
} else if (m instanceof Append) {
table.append((Append) m);
} else if (m instanceof Put) {
table.checkAndPut(row, cf, q, v, (Put) m);
table.checkAndMutate(row, cf).qualifier(q).ifEquals(v).thenPut((Put) m);
} else if (m instanceof Delete) {
table.checkAndDelete(row, cf, q, v, (Delete) m);
table.checkAndMutate(row, cf).qualifier(q).ifEquals(v).thenDelete((Delete) m);
} else {
throw new IllegalArgumentException("unsupported mutation "
+ m.getClass().getSimpleName());

View File

@ -1891,8 +1891,13 @@ public class ThriftServerRunner implements Runnable {
try {
table = getTable(tableName);
byte[][] famAndQf = CellUtil.parseColumn(getBytes(column));
return table.checkAndPut(getBytes(row), famAndQf[0], famAndQf[1],
value != null ? getBytes(value) : HConstants.EMPTY_BYTE_ARRAY, put);
Table.CheckAndMutateBuilder mutateBuilder =
table.checkAndMutate(getBytes(row), famAndQf[0]).qualifier(famAndQf[1]);
if (value != null) {
return mutateBuilder.ifEquals(getBytes(value)).thenPut(put);
} else {
return mutateBuilder.ifNotExists().thenPut(put);
}
} catch (IOException e) {
LOG.warn(e.getMessage(), e);
throw getIOError(e);

View File

@ -317,9 +317,13 @@ public class ThriftHBaseServiceHandler implements THBaseService.Iface {
checkReadOnlyMode();
Table htable = getTable(table);
try {
return htable.checkAndPut(byteBufferToByteArray(row), byteBufferToByteArray(family),
byteBufferToByteArray(qualifier), (value == null) ? null : byteBufferToByteArray(value),
putFromThrift(put));
Table.CheckAndMutateBuilder builder = htable.checkAndMutate(byteBufferToByteArray(row),
byteBufferToByteArray(family)).qualifier(byteBufferToByteArray(qualifier));
if (value == null) {
return builder.ifNotExists().thenPut(putFromThrift(put));
} else {
return builder.ifEquals(byteBufferToByteArray(value)).thenPut(putFromThrift(put));
}
} catch (IOException e) {
throw getTIOError(e);
} finally {
@ -374,9 +378,10 @@ public class ThriftHBaseServiceHandler implements THBaseService.Iface {
throws TIOError, TException {
checkReadOnlyMode();
try (final Table htable = getTable(table)) {
return htable.checkAndMutate(byteBufferToByteArray(row), byteBufferToByteArray(family),
byteBufferToByteArray(qualifier), compareOpFromThrift(compareOp),
byteBufferToByteArray(value), rowMutationsFromThrift(rowMutations));
return htable.checkAndMutate(byteBufferToByteArray(row), byteBufferToByteArray(family))
.qualifier(byteBufferToByteArray(qualifier))
.ifMatches(compareOpFromThrift(compareOp), byteBufferToByteArray(value))
.thenMutate(rowMutationsFromThrift(rowMutations));
} catch (IOException e) {
throw getTIOError(e);
}
@ -388,13 +393,14 @@ public class ThriftHBaseServiceHandler implements THBaseService.Iface {
checkReadOnlyMode();
Table htable = getTable(table);
try {
Table.CheckAndMutateBuilder mutateBuilder =
htable.checkAndMutate(byteBufferToByteArray(row), byteBufferToByteArray(family))
.qualifier(byteBufferToByteArray(qualifier));
if (value == null) {
return htable.checkAndDelete(byteBufferToByteArray(row), byteBufferToByteArray(family),
byteBufferToByteArray(qualifier), null, deleteFromThrift(deleteSingle));
return mutateBuilder.ifNotExists().thenDelete(deleteFromThrift(deleteSingle));
} else {
return htable.checkAndDelete(byteBufferToByteArray(row), byteBufferToByteArray(family),
byteBufferToByteArray(qualifier), byteBufferToByteArray(value),
deleteFromThrift(deleteSingle));
return mutateBuilder.ifEquals(byteBufferToByteArray(value))
.thenDelete(deleteFromThrift(deleteSingle));
}
} catch (IOException e) {
throw getTIOError(e);