001/*
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hbase.client;
019
020import static org.junit.Assert.assertArrayEquals;
021import static org.junit.Assert.assertEquals;
022import static org.junit.Assert.assertFalse;
023import static org.junit.Assert.assertNotNull;
024import static org.junit.Assert.assertNull;
025import static org.junit.Assert.assertTrue;
026import static org.junit.Assert.fail;
027
028import java.io.IOException;
029import java.util.ArrayList;
030import java.util.Arrays;
031import java.util.LinkedList;
032import java.util.List;
033import java.util.Map;
034import java.util.NavigableMap;
035import java.util.concurrent.Callable;
036import java.util.concurrent.ExecutorService;
037import java.util.concurrent.Executors;
038import java.util.concurrent.atomic.AtomicReference;
039import org.apache.commons.lang3.ArrayUtils;
040import org.apache.hadoop.conf.Configuration;
041import org.apache.hadoop.fs.Path;
042import org.apache.hadoop.hbase.Cell;
043import org.apache.hadoop.hbase.CellScanner;
044import org.apache.hadoop.hbase.CellUtil;
045import org.apache.hadoop.hbase.CompareOperator;
046import org.apache.hadoop.hbase.DoNotRetryIOException;
047import org.apache.hadoop.hbase.HBaseClassTestRule;
048import org.apache.hadoop.hbase.HConstants;
049import org.apache.hadoop.hbase.HRegionLocation;
050import org.apache.hadoop.hbase.KeyValue;
051import org.apache.hadoop.hbase.PrivateCellUtil;
052import org.apache.hadoop.hbase.ServerName;
053import org.apache.hadoop.hbase.TableName;
054import org.apache.hadoop.hbase.TableNameTestRule;
055import org.apache.hadoop.hbase.Waiter;
056import org.apache.hadoop.hbase.client.metrics.ScanMetrics;
057import org.apache.hadoop.hbase.coprocessor.MultiRowMutationEndpoint;
058import org.apache.hadoop.hbase.filter.BinaryComparator;
059import org.apache.hadoop.hbase.filter.Filter;
060import org.apache.hadoop.hbase.filter.FilterList;
061import org.apache.hadoop.hbase.filter.FirstKeyOnlyFilter;
062import org.apache.hadoop.hbase.filter.InclusiveStopFilter;
063import org.apache.hadoop.hbase.filter.KeyOnlyFilter;
064import org.apache.hadoop.hbase.filter.QualifierFilter;
065import org.apache.hadoop.hbase.filter.RegexStringComparator;
066import org.apache.hadoop.hbase.filter.RowFilter;
067import org.apache.hadoop.hbase.filter.SingleColumnValueFilter;
068import org.apache.hadoop.hbase.filter.SubstringComparator;
069import org.apache.hadoop.hbase.filter.ValueFilter;
070import org.apache.hadoop.hbase.io.TimeRange;
071import org.apache.hadoop.hbase.io.hfile.BlockCache;
072import org.apache.hadoop.hbase.io.hfile.CacheConfig;
073import org.apache.hadoop.hbase.ipc.CoprocessorRpcChannel;
074import org.apache.hadoop.hbase.protobuf.ProtobufUtil;
075import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.MutationProto;
076import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.MutationProto.MutationType;
077import org.apache.hadoop.hbase.protobuf.generated.MultiRowMutationProtos.MultiRowMutationService;
078import org.apache.hadoop.hbase.protobuf.generated.MultiRowMutationProtos.MutateRowsRequest;
079import org.apache.hadoop.hbase.protobuf.generated.MultiRowMutationProtos.MutateRowsResponse;
080import org.apache.hadoop.hbase.regionserver.HRegion;
081import org.apache.hadoop.hbase.regionserver.HRegionServer;
082import org.apache.hadoop.hbase.regionserver.HStore;
083import org.apache.hadoop.hbase.regionserver.NoSuchColumnFamilyException;
084import org.apache.hadoop.hbase.testclassification.ClientTests;
085import org.apache.hadoop.hbase.testclassification.LargeTests;
086import org.apache.hadoop.hbase.util.Bytes;
087import org.apache.hadoop.hbase.util.CommonFSUtils;
088import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
089import org.apache.hadoop.hbase.util.FSUtils;
090import org.junit.AfterClass;
091import org.junit.ClassRule;
092import org.junit.Ignore;
093import org.junit.Rule;
094import org.junit.Test;
095import org.junit.experimental.categories.Category;
096import org.junit.runner.RunWith;
097import org.junit.runners.Parameterized;
098import org.junit.runners.Parameterized.Parameters;
099import org.slf4j.Logger;
100import org.slf4j.LoggerFactory;
101
102/**
103 * Run tests that use the HBase clients; {@link Table}. Sets up the HBase mini cluster once at start
104 * and runs through all client tests. Each creates a table named for the method and does its stuff
105 * against that. Parameterized to run with different registry implementations.
106 */
107@Category({ LargeTests.class, ClientTests.class })
108@SuppressWarnings("deprecation")
109@RunWith(Parameterized.class)
110public class TestFromClientSide5 extends FromClientSideBase {
111  private static final Logger LOG = LoggerFactory.getLogger(TestFromClientSide5.class);
112
113  @ClassRule
114  public static final HBaseClassTestRule CLASS_RULE =
115    HBaseClassTestRule.forClass(TestFromClientSide5.class);
116
117  @Rule
118  public TableNameTestRule name = new TableNameTestRule();
119
120  // To keep the child classes happy.
121  TestFromClientSide5() {
122  }
123
124  public TestFromClientSide5(Class<? extends ConnectionRegistry> registry, int numHedgedReqs)
125    throws Exception {
126    initialize(registry, numHedgedReqs, MultiRowMutationEndpoint.class);
127  }
128
129  @Parameters(name = "{index}: registry={0}, numHedgedReqs={1}")
130  public static List<Object[]> parameters() {
131    return Arrays.asList(new Object[] { MasterRegistry.class, 1 },
132      new Object[] { MasterRegistry.class, 2 }, new Object[] { ZKConnectionRegistry.class, 1 });
133  }
134
135  @AfterClass
136  public static void tearDownAfterClass() throws Exception {
137    afterClass();
138  }
139
140  @Test
141  public void testGetClosestRowBefore() throws IOException, InterruptedException {
142    final TableName tableName = name.getTableName();
143    final byte[] firstRow = Bytes.toBytes("row111");
144    final byte[] secondRow = Bytes.toBytes("row222");
145    final byte[] thirdRow = Bytes.toBytes("row333");
146    final byte[] forthRow = Bytes.toBytes("row444");
147    final byte[] beforeFirstRow = Bytes.toBytes("row");
148    final byte[] beforeSecondRow = Bytes.toBytes("row22");
149    final byte[] beforeThirdRow = Bytes.toBytes("row33");
150    final byte[] beforeForthRow = Bytes.toBytes("row44");
151
152    try (
153      Table table = TEST_UTIL.createTable(tableName,
154        new byte[][] { HConstants.CATALOG_FAMILY, Bytes.toBytes("info2") }, 1, 1024);
155      RegionLocator locator = TEST_UTIL.getConnection().getRegionLocator(tableName)) {
156
157      // set block size to 64 to making 2 kvs into one block, bypassing the walkForwardInSingleRow
158      // in Store.rowAtOrBeforeFromStoreFile
159      String regionName = locator.getAllRegionLocations().get(0).getRegion().getEncodedName();
160      HRegion region = TEST_UTIL.getRSForFirstRegionInTable(tableName).getRegion(regionName);
161      Put put1 = new Put(firstRow);
162      Put put2 = new Put(secondRow);
163      Put put3 = new Put(thirdRow);
164      Put put4 = new Put(forthRow);
165      byte[] one = new byte[] { 1 };
166      byte[] two = new byte[] { 2 };
167      byte[] three = new byte[] { 3 };
168      byte[] four = new byte[] { 4 };
169
170      put1.addColumn(HConstants.CATALOG_FAMILY, null, one);
171      put2.addColumn(HConstants.CATALOG_FAMILY, null, two);
172      put3.addColumn(HConstants.CATALOG_FAMILY, null, three);
173      put4.addColumn(HConstants.CATALOG_FAMILY, null, four);
174      table.put(put1);
175      table.put(put2);
176      table.put(put3);
177      table.put(put4);
178      region.flush(true);
179
180      Result result;
181
182      // Test before first that null is returned
183      result = getReverseScanResult(table, beforeFirstRow);
184      assertNull(result);
185
186      // Test at first that first is returned
187      result = getReverseScanResult(table, firstRow);
188      assertTrue(result.containsColumn(HConstants.CATALOG_FAMILY, null));
189      assertTrue(Bytes.equals(result.getRow(), firstRow));
190      assertTrue(Bytes.equals(result.getValue(HConstants.CATALOG_FAMILY, null), one));
191
192      // Test in between first and second that first is returned
193      result = getReverseScanResult(table, beforeSecondRow);
194      assertTrue(result.containsColumn(HConstants.CATALOG_FAMILY, null));
195      assertTrue(Bytes.equals(result.getRow(), firstRow));
196      assertTrue(Bytes.equals(result.getValue(HConstants.CATALOG_FAMILY, null), one));
197
198      // Test at second make sure second is returned
199      result = getReverseScanResult(table, secondRow);
200      assertTrue(result.containsColumn(HConstants.CATALOG_FAMILY, null));
201      assertTrue(Bytes.equals(result.getRow(), secondRow));
202      assertTrue(Bytes.equals(result.getValue(HConstants.CATALOG_FAMILY, null), two));
203
204      // Test in second and third, make sure second is returned
205      result = getReverseScanResult(table, beforeThirdRow);
206      assertTrue(result.containsColumn(HConstants.CATALOG_FAMILY, null));
207      assertTrue(Bytes.equals(result.getRow(), secondRow));
208      assertTrue(Bytes.equals(result.getValue(HConstants.CATALOG_FAMILY, null), two));
209
210      // Test at third make sure third is returned
211      result = getReverseScanResult(table, thirdRow);
212      assertTrue(result.containsColumn(HConstants.CATALOG_FAMILY, null));
213      assertTrue(Bytes.equals(result.getRow(), thirdRow));
214      assertTrue(Bytes.equals(result.getValue(HConstants.CATALOG_FAMILY, null), three));
215
216      // Test in third and forth, make sure third is returned
217      result = getReverseScanResult(table, beforeForthRow);
218      assertTrue(result.containsColumn(HConstants.CATALOG_FAMILY, null));
219      assertTrue(Bytes.equals(result.getRow(), thirdRow));
220      assertTrue(Bytes.equals(result.getValue(HConstants.CATALOG_FAMILY, null), three));
221
222      // Test at forth make sure forth is returned
223      result = getReverseScanResult(table, forthRow);
224      assertTrue(result.containsColumn(HConstants.CATALOG_FAMILY, null));
225      assertTrue(Bytes.equals(result.getRow(), forthRow));
226      assertTrue(Bytes.equals(result.getValue(HConstants.CATALOG_FAMILY, null), four));
227
228      // Test after forth make sure forth is returned
229      result = getReverseScanResult(table, Bytes.add(forthRow, one));
230      assertTrue(result.containsColumn(HConstants.CATALOG_FAMILY, null));
231      assertTrue(Bytes.equals(result.getRow(), forthRow));
232      assertTrue(Bytes.equals(result.getValue(HConstants.CATALOG_FAMILY, null), four));
233    }
234  }
235
236  private Result getReverseScanResult(Table table, byte[] row) throws IOException {
237    Scan scan = new Scan(row);
238    scan.setSmall(true);
239    scan.setReversed(true);
240    scan.setCaching(1);
241    scan.addFamily(HConstants.CATALOG_FAMILY);
242    try (ResultScanner scanner = table.getScanner(scan)) {
243      return scanner.next();
244    }
245  }
246
247  /**
248   * For HBASE-2156
249   */
250  @Test
251  public void testScanVariableReuse() {
252    Scan scan = new Scan();
253    scan.addFamily(FAMILY);
254    scan.addColumn(FAMILY, ROW);
255
256    assertEquals(1, scan.getFamilyMap().get(FAMILY).size());
257
258    scan = new Scan();
259    scan.addFamily(FAMILY);
260
261    assertNull(scan.getFamilyMap().get(FAMILY));
262    assertTrue(scan.getFamilyMap().containsKey(FAMILY));
263  }
264
265  @Test
266  public void testMultiRowMutation() throws Exception {
267    LOG.info("Starting testMultiRowMutation");
268    final TableName tableName = name.getTableName();
269    final byte[] ROW1 = Bytes.toBytes("testRow1");
270    final byte[] ROW2 = Bytes.toBytes("testRow2");
271    final byte[] ROW3 = Bytes.toBytes("testRow3");
272
273    try (Table t = TEST_UTIL.createTable(tableName, FAMILY)) {
274      // Add initial data
275      t.batch(Arrays.asList(new Put(ROW1).addColumn(FAMILY, QUALIFIER, VALUE),
276        new Put(ROW2).addColumn(FAMILY, QUALIFIER, Bytes.toBytes(1L)),
277        new Put(ROW3).addColumn(FAMILY, QUALIFIER, VALUE)), new Object[3]);
278
279      // Execute MultiRowMutation
280      Put put = new Put(ROW).addColumn(FAMILY, QUALIFIER, VALUE);
281      MutationProto m1 = ProtobufUtil.toMutation(MutationType.PUT, put);
282
283      Delete delete = new Delete(ROW1);
284      MutationProto m2 = ProtobufUtil.toMutation(MutationType.DELETE, delete);
285
286      Increment increment = new Increment(ROW2).addColumn(FAMILY, QUALIFIER, 1L);
287      MutationProto m3 = ProtobufUtil.toMutation(MutationType.INCREMENT, increment);
288
289      Append append = new Append(ROW3).addColumn(FAMILY, QUALIFIER, VALUE);
290      MutationProto m4 = ProtobufUtil.toMutation(MutationType.APPEND, append);
291
292      MutateRowsRequest.Builder mrmBuilder = MutateRowsRequest.newBuilder();
293      mrmBuilder.addMutationRequest(m1);
294      mrmBuilder.addMutationRequest(m2);
295      mrmBuilder.addMutationRequest(m3);
296      mrmBuilder.addMutationRequest(m4);
297
298      CoprocessorRpcChannel channel = t.coprocessorService(ROW);
299      MultiRowMutationService.BlockingInterface service =
300        MultiRowMutationService.newBlockingStub(channel);
301      MutateRowsResponse response = service.mutateRows(null, mrmBuilder.build());
302
303      // Assert
304      assertTrue(response.getProcessed());
305
306      Result r = t.get(new Get(ROW));
307      assertEquals(Bytes.toString(VALUE), Bytes.toString(r.getValue(FAMILY, QUALIFIER)));
308
309      r = t.get(new Get(ROW1));
310      assertTrue(r.isEmpty());
311
312      r = t.get(new Get(ROW2));
313      assertEquals(2L, Bytes.toLong(r.getValue(FAMILY, QUALIFIER)));
314
315      r = t.get(new Get(ROW3));
316      assertEquals(Bytes.toString(VALUE) + Bytes.toString(VALUE),
317        Bytes.toString(r.getValue(FAMILY, QUALIFIER)));
318    }
319  }
320
321  @Test
322  public void testMultiRowMutationWithSingleConditionWhenConditionMatches() throws Exception {
323    final TableName tableName = name.getTableName();
324    final byte[] ROW1 = Bytes.toBytes("testRow1");
325    final byte[] ROW2 = Bytes.toBytes("testRow2");
326    final byte[] VALUE1 = Bytes.toBytes("testValue1");
327    final byte[] VALUE2 = Bytes.toBytes("testValue2");
328
329    try (Table t = TEST_UTIL.createTable(tableName, FAMILY)) {
330      // Add initial data
331      t.put(new Put(ROW2).addColumn(FAMILY, QUALIFIER, VALUE2));
332
333      // Execute MultiRowMutation with conditions
334      Put put1 = new Put(ROW).addColumn(FAMILY, QUALIFIER, VALUE);
335      MutationProto m1 = ProtobufUtil.toMutation(MutationType.PUT, put1);
336      Put put2 = new Put(ROW1).addColumn(FAMILY, QUALIFIER, VALUE1);
337      MutationProto m2 = ProtobufUtil.toMutation(MutationType.PUT, put2);
338      Delete delete = new Delete(ROW2);
339      MutationProto m3 = ProtobufUtil.toMutation(MutationType.DELETE, delete);
340
341      MutateRowsRequest.Builder mrmBuilder = MutateRowsRequest.newBuilder();
342      mrmBuilder.addMutationRequest(m1);
343      mrmBuilder.addMutationRequest(m2);
344      mrmBuilder.addMutationRequest(m3);
345      mrmBuilder.addCondition(
346        ProtobufUtil.toCondition(ROW2, FAMILY, QUALIFIER, CompareOperator.EQUAL, VALUE2, null));
347
348      CoprocessorRpcChannel channel = t.coprocessorService(ROW);
349      MultiRowMutationService.BlockingInterface service =
350        MultiRowMutationService.newBlockingStub(channel);
351      MutateRowsResponse response = service.mutateRows(null, mrmBuilder.build());
352
353      // Assert
354      assertTrue(response.getProcessed());
355
356      Result r = t.get(new Get(ROW));
357      assertEquals(Bytes.toString(VALUE), Bytes.toString(r.getValue(FAMILY, QUALIFIER)));
358
359      r = t.get(new Get(ROW1));
360      assertEquals(Bytes.toString(VALUE1), Bytes.toString(r.getValue(FAMILY, QUALIFIER)));
361
362      r = t.get(new Get(ROW2));
363      assertTrue(r.isEmpty());
364    }
365  }
366
367  @Test
368  public void testMultiRowMutationWithSingleConditionWhenConditionNotMatch() throws Exception {
369    final TableName tableName = name.getTableName();
370    final byte[] ROW1 = Bytes.toBytes("testRow1");
371    final byte[] ROW2 = Bytes.toBytes("testRow2");
372    final byte[] VALUE1 = Bytes.toBytes("testValue1");
373    final byte[] VALUE2 = Bytes.toBytes("testValue2");
374
375    try (Table t = TEST_UTIL.createTable(tableName, FAMILY)) {
376      // Add initial data
377      t.put(new Put(ROW2).addColumn(FAMILY, QUALIFIER, VALUE2));
378
379      // Execute MultiRowMutation with conditions
380      Put put1 = new Put(ROW).addColumn(FAMILY, QUALIFIER, VALUE);
381      MutationProto m1 = ProtobufUtil.toMutation(MutationType.PUT, put1);
382      Put put2 = new Put(ROW1).addColumn(FAMILY, QUALIFIER, VALUE1);
383      MutationProto m2 = ProtobufUtil.toMutation(MutationType.PUT, put2);
384      Delete delete = new Delete(ROW2);
385      MutationProto m3 = ProtobufUtil.toMutation(MutationType.DELETE, delete);
386
387      MutateRowsRequest.Builder mrmBuilder = MutateRowsRequest.newBuilder();
388      mrmBuilder.addMutationRequest(m1);
389      mrmBuilder.addMutationRequest(m2);
390      mrmBuilder.addMutationRequest(m3);
391      mrmBuilder.addCondition(
392        ProtobufUtil.toCondition(ROW2, FAMILY, QUALIFIER, CompareOperator.EQUAL, VALUE1, null));
393
394      CoprocessorRpcChannel channel = t.coprocessorService(ROW);
395      MultiRowMutationService.BlockingInterface service =
396        MultiRowMutationService.newBlockingStub(channel);
397      MutateRowsResponse response = service.mutateRows(null, mrmBuilder.build());
398
399      // Assert
400      assertFalse(response.getProcessed());
401
402      Result r = t.get(new Get(ROW));
403      assertTrue(r.isEmpty());
404
405      r = t.get(new Get(ROW1));
406      assertTrue(r.isEmpty());
407
408      r = t.get(new Get(ROW2));
409      assertEquals(Bytes.toString(VALUE2), Bytes.toString(r.getValue(FAMILY, QUALIFIER)));
410    }
411  }
412
413  @Test
414  public void testMultiRowMutationWithMultipleConditionsWhenConditionsMatch() throws Exception {
415    final TableName tableName = name.getTableName();
416    final byte[] ROW1 = Bytes.toBytes("testRow1");
417    final byte[] ROW2 = Bytes.toBytes("testRow2");
418    final byte[] VALUE1 = Bytes.toBytes("testValue1");
419    final byte[] VALUE2 = Bytes.toBytes("testValue2");
420
421    try (Table t = TEST_UTIL.createTable(tableName, FAMILY)) {
422      // Add initial data
423      t.put(new Put(ROW2).addColumn(FAMILY, QUALIFIER, VALUE2));
424
425      // Execute MultiRowMutation with conditions
426      Put put1 = new Put(ROW).addColumn(FAMILY, QUALIFIER, VALUE);
427      MutationProto m1 = ProtobufUtil.toMutation(MutationType.PUT, put1);
428      Put put2 = new Put(ROW1).addColumn(FAMILY, QUALIFIER, VALUE1);
429      MutationProto m2 = ProtobufUtil.toMutation(MutationType.PUT, put2);
430      Delete delete = new Delete(ROW2);
431      MutationProto m3 = ProtobufUtil.toMutation(MutationType.DELETE, delete);
432
433      MutateRowsRequest.Builder mrmBuilder = MutateRowsRequest.newBuilder();
434      mrmBuilder.addMutationRequest(m1);
435      mrmBuilder.addMutationRequest(m2);
436      mrmBuilder.addMutationRequest(m3);
437      mrmBuilder.addCondition(
438        ProtobufUtil.toCondition(ROW, FAMILY, QUALIFIER, CompareOperator.EQUAL, null, null));
439      mrmBuilder.addCondition(
440        ProtobufUtil.toCondition(ROW2, FAMILY, QUALIFIER, CompareOperator.EQUAL, VALUE2, null));
441
442      CoprocessorRpcChannel channel = t.coprocessorService(ROW);
443      MultiRowMutationService.BlockingInterface service =
444        MultiRowMutationService.newBlockingStub(channel);
445      MutateRowsResponse response = service.mutateRows(null, mrmBuilder.build());
446
447      // Assert
448      assertTrue(response.getProcessed());
449
450      Result r = t.get(new Get(ROW));
451      assertEquals(Bytes.toString(VALUE), Bytes.toString(r.getValue(FAMILY, QUALIFIER)));
452
453      r = t.get(new Get(ROW1));
454      assertEquals(Bytes.toString(VALUE1), Bytes.toString(r.getValue(FAMILY, QUALIFIER)));
455
456      r = t.get(new Get(ROW2));
457      assertTrue(r.isEmpty());
458    }
459  }
460
461  @Test
462  public void testMultiRowMutationWithMultipleConditionsWhenConditionsNotMatch() throws Exception {
463    final TableName tableName = name.getTableName();
464    final byte[] ROW1 = Bytes.toBytes("testRow1");
465    final byte[] ROW2 = Bytes.toBytes("testRow2");
466    final byte[] VALUE1 = Bytes.toBytes("testValue1");
467    final byte[] VALUE2 = Bytes.toBytes("testValue2");
468
469    try (Table t = TEST_UTIL.createTable(tableName, FAMILY)) {
470      // Add initial data
471      t.put(new Put(ROW2).addColumn(FAMILY, QUALIFIER, VALUE2));
472
473      // Execute MultiRowMutation with conditions
474      Put put1 = new Put(ROW).addColumn(FAMILY, QUALIFIER, VALUE);
475      MutationProto m1 = ProtobufUtil.toMutation(MutationType.PUT, put1);
476      Put put2 = new Put(ROW1).addColumn(FAMILY, QUALIFIER, VALUE1);
477      MutationProto m2 = ProtobufUtil.toMutation(MutationType.PUT, put2);
478      Delete delete = new Delete(ROW2);
479      MutationProto m3 = ProtobufUtil.toMutation(MutationType.DELETE, delete);
480
481      MutateRowsRequest.Builder mrmBuilder = MutateRowsRequest.newBuilder();
482      mrmBuilder.addMutationRequest(m1);
483      mrmBuilder.addMutationRequest(m2);
484      mrmBuilder.addMutationRequest(m3);
485      mrmBuilder.addCondition(
486        ProtobufUtil.toCondition(ROW1, FAMILY, QUALIFIER, CompareOperator.EQUAL, null, null));
487      mrmBuilder.addCondition(
488        ProtobufUtil.toCondition(ROW2, FAMILY, QUALIFIER, CompareOperator.EQUAL, VALUE1, null));
489
490      CoprocessorRpcChannel channel = t.coprocessorService(ROW);
491      MultiRowMutationService.BlockingInterface service =
492        MultiRowMutationService.newBlockingStub(channel);
493      MutateRowsResponse response = service.mutateRows(null, mrmBuilder.build());
494
495      // Assert
496      assertFalse(response.getProcessed());
497
498      Result r = t.get(new Get(ROW));
499      assertTrue(r.isEmpty());
500
501      r = t.get(new Get(ROW1));
502      assertTrue(r.isEmpty());
503
504      r = t.get(new Get(ROW2));
505      assertEquals(Bytes.toString(VALUE2), Bytes.toString(r.getValue(FAMILY, QUALIFIER)));
506    }
507  }
508
509  @Test
510  public void testMultiRowMutationWithFilterConditionWhenConditionMatches() throws Exception {
511    final TableName tableName = name.getTableName();
512    final byte[] ROW1 = Bytes.toBytes("testRow1");
513    final byte[] ROW2 = Bytes.toBytes("testRow2");
514    final byte[] QUALIFIER2 = Bytes.toBytes("testQualifier2");
515    final byte[] VALUE1 = Bytes.toBytes("testValue1");
516    final byte[] VALUE2 = Bytes.toBytes("testValue2");
517    final byte[] VALUE3 = Bytes.toBytes("testValue3");
518
519    try (Table t = TEST_UTIL.createTable(tableName, FAMILY)) {
520      // Add initial data
521      t.put(
522        new Put(ROW2).addColumn(FAMILY, QUALIFIER, VALUE2).addColumn(FAMILY, QUALIFIER2, VALUE3));
523
524      // Execute MultiRowMutation with conditions
525      Put put1 = new Put(ROW).addColumn(FAMILY, QUALIFIER, VALUE);
526      MutationProto m1 = ProtobufUtil.toMutation(MutationType.PUT, put1);
527      Put put2 = new Put(ROW1).addColumn(FAMILY, QUALIFIER, VALUE1);
528      MutationProto m2 = ProtobufUtil.toMutation(MutationType.PUT, put2);
529      Delete delete = new Delete(ROW2);
530      MutationProto m3 = ProtobufUtil.toMutation(MutationType.DELETE, delete);
531
532      MutateRowsRequest.Builder mrmBuilder = MutateRowsRequest.newBuilder();
533      mrmBuilder.addMutationRequest(m1);
534      mrmBuilder.addMutationRequest(m2);
535      mrmBuilder.addMutationRequest(m3);
536      mrmBuilder.addCondition(ProtobufUtil.toCondition(ROW2,
537        new FilterList(
538          new SingleColumnValueFilter(FAMILY, QUALIFIER, CompareOperator.EQUAL, VALUE2),
539          new SingleColumnValueFilter(FAMILY, QUALIFIER2, CompareOperator.EQUAL, VALUE3)),
540        null));
541
542      CoprocessorRpcChannel channel = t.coprocessorService(ROW);
543      MultiRowMutationService.BlockingInterface service =
544        MultiRowMutationService.newBlockingStub(channel);
545      MutateRowsResponse response = service.mutateRows(null, mrmBuilder.build());
546
547      // Assert
548      assertTrue(response.getProcessed());
549
550      Result r = t.get(new Get(ROW));
551      assertEquals(Bytes.toString(VALUE), Bytes.toString(r.getValue(FAMILY, QUALIFIER)));
552
553      r = t.get(new Get(ROW1));
554      assertEquals(Bytes.toString(VALUE1), Bytes.toString(r.getValue(FAMILY, QUALIFIER)));
555
556      r = t.get(new Get(ROW2));
557      assertTrue(r.isEmpty());
558    }
559  }
560
561  @Test
562  public void testMultiRowMutationWithFilterConditionWhenConditionNotMatch() throws Exception {
563    final TableName tableName = name.getTableName();
564    final byte[] ROW1 = Bytes.toBytes("testRow1");
565    final byte[] ROW2 = Bytes.toBytes("testRow2");
566    final byte[] QUALIFIER2 = Bytes.toBytes("testQualifier2");
567    final byte[] VALUE1 = Bytes.toBytes("testValue1");
568    final byte[] VALUE2 = Bytes.toBytes("testValue2");
569    final byte[] VALUE3 = Bytes.toBytes("testValue3");
570
571    try (Table t = TEST_UTIL.createTable(tableName, FAMILY)) {
572      // Add initial data
573      t.put(
574        new Put(ROW2).addColumn(FAMILY, QUALIFIER, VALUE2).addColumn(FAMILY, QUALIFIER2, VALUE3));
575
576      // Execute MultiRowMutation with conditions
577      Put put1 = new Put(ROW).addColumn(FAMILY, QUALIFIER, VALUE);
578      MutationProto m1 = ProtobufUtil.toMutation(MutationType.PUT, put1);
579      Put put2 = new Put(ROW1).addColumn(FAMILY, QUALIFIER, VALUE1);
580      MutationProto m2 = ProtobufUtil.toMutation(MutationType.PUT, put2);
581      Delete delete = new Delete(ROW2);
582      MutationProto m3 = ProtobufUtil.toMutation(MutationType.DELETE, delete);
583
584      MutateRowsRequest.Builder mrmBuilder = MutateRowsRequest.newBuilder();
585      mrmBuilder.addMutationRequest(m1);
586      mrmBuilder.addMutationRequest(m2);
587      mrmBuilder.addMutationRequest(m3);
588      mrmBuilder.addCondition(ProtobufUtil.toCondition(ROW2,
589        new FilterList(
590          new SingleColumnValueFilter(FAMILY, QUALIFIER, CompareOperator.EQUAL, VALUE2),
591          new SingleColumnValueFilter(FAMILY, QUALIFIER2, CompareOperator.EQUAL, VALUE2)),
592        null));
593
594      CoprocessorRpcChannel channel = t.coprocessorService(ROW);
595      MultiRowMutationService.BlockingInterface service =
596        MultiRowMutationService.newBlockingStub(channel);
597      MutateRowsResponse response = service.mutateRows(null, mrmBuilder.build());
598
599      // Assert
600      assertFalse(response.getProcessed());
601
602      Result r = t.get(new Get(ROW));
603      assertTrue(r.isEmpty());
604
605      r = t.get(new Get(ROW1));
606      assertTrue(r.isEmpty());
607
608      r = t.get(new Get(ROW2));
609      assertEquals(Bytes.toString(VALUE2), Bytes.toString(r.getValue(FAMILY, QUALIFIER)));
610    }
611  }
612
613  @Test
614  public void testRowMutations() throws Exception {
615    LOG.info("Starting testRowMutations");
616    final TableName tableName = name.getTableName();
617    try (Table t = TEST_UTIL.createTable(tableName, FAMILY)) {
618      byte[][] QUALIFIERS = new byte[][] { Bytes.toBytes("a"), Bytes.toBytes("b"),
619        Bytes.toBytes("c"), Bytes.toBytes("d") };
620
621      // Test for Put operations
622      RowMutations arm = new RowMutations(ROW);
623      Put p = new Put(ROW);
624      p.addColumn(FAMILY, QUALIFIERS[0], VALUE);
625      arm.add(p);
626      Result r = t.mutateRow(arm);
627      assertTrue(r.getExists());
628      assertTrue(r.isEmpty());
629
630      Get g = new Get(ROW);
631      r = t.get(g);
632      assertEquals(0, Bytes.compareTo(VALUE, r.getValue(FAMILY, QUALIFIERS[0])));
633
634      // Test for Put and Delete operations
635      arm = new RowMutations(ROW);
636      p = new Put(ROW);
637      p.addColumn(FAMILY, QUALIFIERS[1], VALUE);
638      arm.add(p);
639      Delete d = new Delete(ROW);
640      d.addColumns(FAMILY, QUALIFIERS[0]);
641      arm.add(d);
642      // TODO: Trying mutateRow again. The batch was failing with a one try only.
643      r = t.mutateRow(arm);
644      assertTrue(r.getExists());
645      assertTrue(r.isEmpty());
646
647      r = t.get(g);
648      assertEquals(0, Bytes.compareTo(VALUE, r.getValue(FAMILY, QUALIFIERS[1])));
649      assertNull(r.getValue(FAMILY, QUALIFIERS[0]));
650
651      // Test for Increment and Append operations
652      arm = new RowMutations(ROW);
653      arm.add(Arrays.asList(new Put(ROW).addColumn(FAMILY, QUALIFIERS[0], VALUE),
654        new Delete(ROW).addColumns(FAMILY, QUALIFIERS[1]),
655        new Increment(ROW).addColumn(FAMILY, QUALIFIERS[2], 5L),
656        new Append(ROW).addColumn(FAMILY, QUALIFIERS[3], Bytes.toBytes("abc"))));
657      r = t.mutateRow(arm);
658      assertTrue(r.getExists());
659      assertEquals(5L, Bytes.toLong(r.getValue(FAMILY, QUALIFIERS[2])));
660      assertEquals("abc", Bytes.toString(r.getValue(FAMILY, QUALIFIERS[3])));
661
662      g = new Get(ROW);
663      r = t.get(g);
664      assertEquals(0, Bytes.compareTo(VALUE, r.getValue(FAMILY, QUALIFIERS[0])));
665      assertNull(r.getValue(FAMILY, QUALIFIERS[1]));
666      assertEquals(5L, Bytes.toLong(r.getValue(FAMILY, QUALIFIERS[2])));
667      assertEquals("abc", Bytes.toString(r.getValue(FAMILY, QUALIFIERS[3])));
668
669      // Test that we get a region level exception
670      try {
671        arm = new RowMutations(ROW);
672        p = new Put(ROW);
673        p.addColumn(new byte[] { 'b', 'o', 'g', 'u', 's' }, QUALIFIERS[0], VALUE);
674        arm.add(p);
675        t.mutateRow(arm);
676        fail("Expected NoSuchColumnFamilyException");
677      } catch (NoSuchColumnFamilyException e) {
678        return;
679      } catch (RetriesExhaustedWithDetailsException e) {
680        for (Throwable rootCause : e.getCauses()) {
681          if (rootCause instanceof NoSuchColumnFamilyException) {
682            return;
683          }
684        }
685        throw e;
686      }
687    }
688  }
689
690  @Test
691  public void testBatchAppendWithReturnResultFalse() throws Exception {
692    LOG.info("Starting testBatchAppendWithReturnResultFalse");
693    final TableName tableName = name.getTableName();
694    try (Table table = TEST_UTIL.createTable(tableName, FAMILY)) {
695      Append append1 = new Append(Bytes.toBytes("row1"));
696      append1.setReturnResults(false);
697      append1.addColumn(FAMILY, Bytes.toBytes("f1"), Bytes.toBytes("value1"));
698      Append append2 = new Append(Bytes.toBytes("row1"));
699      append2.setReturnResults(false);
700      append2.addColumn(FAMILY, Bytes.toBytes("f1"), Bytes.toBytes("value2"));
701      List<Append> appends = new ArrayList<>();
702      appends.add(append1);
703      appends.add(append2);
704      Object[] results = new Object[2];
705      table.batch(appends, results);
706      assertEquals(2, results.length);
707      for (Object r : results) {
708        Result result = (Result) r;
709        assertTrue(result.isEmpty());
710      }
711    }
712  }
713
714  @Test
715  public void testAppend() throws Exception {
716    LOG.info("Starting testAppend");
717    final TableName tableName = name.getTableName();
718    try (Table t = TEST_UTIL.createTable(tableName, FAMILY)) {
719      byte[] v1 = Bytes.toBytes("42");
720      byte[] v2 = Bytes.toBytes("23");
721      byte[][] QUALIFIERS =
722        new byte[][] { Bytes.toBytes("b"), Bytes.toBytes("a"), Bytes.toBytes("c") };
723      Append a = new Append(ROW);
724      a.addColumn(FAMILY, QUALIFIERS[0], v1);
725      a.addColumn(FAMILY, QUALIFIERS[1], v2);
726      a.setReturnResults(false);
727      assertEmptyResult(t.append(a));
728
729      a = new Append(ROW);
730      a.addColumn(FAMILY, QUALIFIERS[0], v2);
731      a.addColumn(FAMILY, QUALIFIERS[1], v1);
732      a.addColumn(FAMILY, QUALIFIERS[2], v2);
733      Result r = t.append(a);
734      assertEquals(0, Bytes.compareTo(Bytes.add(v1, v2), r.getValue(FAMILY, QUALIFIERS[0])));
735      assertEquals(0, Bytes.compareTo(Bytes.add(v2, v1), r.getValue(FAMILY, QUALIFIERS[1])));
736      // QUALIFIERS[2] previously not exist, verify both value and timestamp are correct
737      assertEquals(0, Bytes.compareTo(v2, r.getValue(FAMILY, QUALIFIERS[2])));
738      assertEquals(r.getColumnLatestCell(FAMILY, QUALIFIERS[0]).getTimestamp(),
739        r.getColumnLatestCell(FAMILY, QUALIFIERS[2]).getTimestamp());
740    }
741  }
742
743  private List<Result> doAppend(final boolean walUsed) throws IOException {
744    LOG.info("Starting testAppend, walUsed is " + walUsed);
745    final TableName TABLENAME =
746      TableName.valueOf(walUsed ? "testAppendWithWAL" : "testAppendWithoutWAL");
747    try (Table t = TEST_UTIL.createTable(TABLENAME, FAMILY)) {
748      final byte[] row1 = Bytes.toBytes("c");
749      final byte[] row2 = Bytes.toBytes("b");
750      final byte[] row3 = Bytes.toBytes("a");
751      final byte[] qual = Bytes.toBytes("qual");
752      Put put_0 = new Put(row2);
753      put_0.addColumn(FAMILY, qual, Bytes.toBytes("put"));
754      Put put_1 = new Put(row3);
755      put_1.addColumn(FAMILY, qual, Bytes.toBytes("put"));
756      Append append_0 = new Append(row1);
757      append_0.addColumn(FAMILY, qual, Bytes.toBytes("i"));
758      Append append_1 = new Append(row1);
759      append_1.addColumn(FAMILY, qual, Bytes.toBytes("k"));
760      Append append_2 = new Append(row1);
761      append_2.addColumn(FAMILY, qual, Bytes.toBytes("e"));
762      if (!walUsed) {
763        append_2.setDurability(Durability.SKIP_WAL);
764      }
765      Append append_3 = new Append(row1);
766      append_3.addColumn(FAMILY, qual, Bytes.toBytes("a"));
767      Scan s = new Scan();
768      s.setCaching(1);
769      t.append(append_0);
770      t.put(put_0);
771      t.put(put_1);
772      List<Result> results = new LinkedList<>();
773      try (ResultScanner scanner = t.getScanner(s)) {
774        // get one row(should be row3) from the scanner to make sure that we have send a request to
775        // region server, which means we have already set the read point, so later we should not see
776        // the new appended values.
777        Result r = scanner.next();
778        assertNotNull(r);
779        results.add(r);
780        t.append(append_1);
781        t.append(append_2);
782        t.append(append_3);
783        for (;;) {
784          r = scanner.next();
785          if (r == null) {
786            break;
787          }
788          results.add(r);
789        }
790      }
791      TEST_UTIL.deleteTable(TABLENAME);
792      return results;
793    }
794  }
795
796  @Test
797  public void testAppendWithoutWAL() throws Exception {
798    List<Result> resultsWithWal = doAppend(true);
799    List<Result> resultsWithoutWal = doAppend(false);
800    assertEquals(resultsWithWal.size(), resultsWithoutWal.size());
801    for (int i = 0; i < resultsWithWal.size(); ++i) {
802      Result resultWithWal = resultsWithWal.get(i);
803      Result resultWithoutWal = resultsWithoutWal.get(i);
804      assertEquals(resultWithWal.rawCells().length, resultWithoutWal.rawCells().length);
805      for (int j = 0; j < resultWithWal.rawCells().length; ++j) {
806        Cell cellWithWal = resultWithWal.rawCells()[j];
807        Cell cellWithoutWal = resultWithoutWal.rawCells()[j];
808        assertArrayEquals(CellUtil.cloneRow(cellWithWal), CellUtil.cloneRow(cellWithoutWal));
809        assertArrayEquals(CellUtil.cloneFamily(cellWithWal), CellUtil.cloneFamily(cellWithoutWal));
810        assertArrayEquals(CellUtil.cloneQualifier(cellWithWal),
811          CellUtil.cloneQualifier(cellWithoutWal));
812        assertArrayEquals(CellUtil.cloneValue(cellWithWal), CellUtil.cloneValue(cellWithoutWal));
813      }
814    }
815  }
816
817  @Test
818  public void testClientPoolRoundRobin() throws IOException {
819    final TableName tableName = name.getTableName();
820
821    int poolSize = 3;
822    int numVersions = poolSize * 2;
823    Configuration conf = TEST_UTIL.getConfiguration();
824    conf.set(HConstants.HBASE_CLIENT_IPC_POOL_TYPE, "round-robin");
825    conf.setInt(HConstants.HBASE_CLIENT_IPC_POOL_SIZE, poolSize);
826
827    try (
828      Table table = TEST_UTIL.createTable(tableName, new byte[][] { FAMILY }, Integer.MAX_VALUE)) {
829
830      final long ts = EnvironmentEdgeManager.currentTime();
831      Get get = new Get(ROW);
832      get.addColumn(FAMILY, QUALIFIER);
833      get.readAllVersions();
834
835      for (int versions = 1; versions <= numVersions; versions++) {
836        Put put = new Put(ROW);
837        put.addColumn(FAMILY, QUALIFIER, ts + versions, VALUE);
838        table.put(put);
839
840        Result result = table.get(get);
841        NavigableMap<Long, byte[]> navigableMap = result.getMap().get(FAMILY).get(QUALIFIER);
842
843        assertEquals("The number of versions of '" + Bytes.toString(FAMILY) + ":"
844          + Bytes.toString(QUALIFIER) + " did not match", versions, navigableMap.size());
845        for (Map.Entry<Long, byte[]> entry : navigableMap.entrySet()) {
846          assertTrue("The value at time " + entry.getKey() + " did not match what was put",
847            Bytes.equals(VALUE, entry.getValue()));
848        }
849      }
850    }
851  }
852
853  @Ignore("Flakey: HBASE-8989")
854  @Test
855  public void testClientPoolThreadLocal() throws IOException {
856    final TableName tableName = name.getTableName();
857
858    int poolSize = Integer.MAX_VALUE;
859    int numVersions = 3;
860    Configuration conf = TEST_UTIL.getConfiguration();
861    conf.set(HConstants.HBASE_CLIENT_IPC_POOL_TYPE, "thread-local");
862    conf.setInt(HConstants.HBASE_CLIENT_IPC_POOL_SIZE, poolSize);
863
864    try (final Table table = TEST_UTIL.createTable(tableName, new byte[][] { FAMILY }, 3)) {
865
866      final long ts = EnvironmentEdgeManager.currentTime();
867      final Get get = new Get(ROW);
868      get.addColumn(FAMILY, QUALIFIER);
869      get.readAllVersions();
870
871      for (int versions = 1; versions <= numVersions; versions++) {
872        Put put = new Put(ROW);
873        put.addColumn(FAMILY, QUALIFIER, ts + versions, VALUE);
874        table.put(put);
875
876        Result result = table.get(get);
877        NavigableMap<Long, byte[]> navigableMap = result.getMap().get(FAMILY).get(QUALIFIER);
878
879        assertEquals("The number of versions of '" + Bytes.toString(FAMILY) + ":"
880          + Bytes.toString(QUALIFIER) + " did not match", versions, navigableMap.size());
881        for (Map.Entry<Long, byte[]> entry : navigableMap.entrySet()) {
882          assertTrue("The value at time " + entry.getKey() + " did not match what was put",
883            Bytes.equals(VALUE, entry.getValue()));
884        }
885      }
886
887      final Object waitLock = new Object();
888      ExecutorService executorService = Executors.newFixedThreadPool(numVersions);
889      final AtomicReference<AssertionError> error = new AtomicReference<>(null);
890      for (int versions = numVersions; versions < numVersions * 2; versions++) {
891        final int versionsCopy = versions;
892        executorService.submit((Callable<Void>) () -> {
893          try {
894            Put put = new Put(ROW);
895            put.addColumn(FAMILY, QUALIFIER, ts + versionsCopy, VALUE);
896            table.put(put);
897
898            Result result = table.get(get);
899            NavigableMap<Long, byte[]> navigableMap = result.getMap().get(FAMILY).get(QUALIFIER);
900
901            assertEquals(
902              "The number of versions of '" + Bytes.toString(FAMILY) + ":"
903                + Bytes.toString(QUALIFIER) + " did not match " + versionsCopy,
904              versionsCopy, navigableMap.size());
905            for (Map.Entry<Long, byte[]> entry : navigableMap.entrySet()) {
906              assertTrue("The value at time " + entry.getKey() + " did not match what was put",
907                Bytes.equals(VALUE, entry.getValue()));
908            }
909            synchronized (waitLock) {
910              waitLock.wait();
911            }
912          } catch (Exception ignored) {
913          } catch (AssertionError e) {
914            // the error happens in a thread, it won't fail the test,
915            // need to pass it to the caller for proper handling.
916            error.set(e);
917            LOG.error(e.toString(), e);
918          }
919
920          return null;
921        });
922      }
923      synchronized (waitLock) {
924        waitLock.notifyAll();
925      }
926      executorService.shutdownNow();
927      assertNull(error.get());
928    }
929  }
930
931  @Test
932  public void testCheckAndPut() throws IOException {
933    final byte[] anotherrow = Bytes.toBytes("anotherrow");
934    final byte[] value2 = Bytes.toBytes("abcd");
935
936    try (Table table = TEST_UTIL.createTable(name.getTableName(), FAMILY)) {
937      Put put1 = new Put(ROW);
938      put1.addColumn(FAMILY, QUALIFIER, VALUE);
939
940      // row doesn't exist, so using non-null value should be considered "not match".
941      boolean ok =
942        table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER).ifEquals(VALUE).thenPut(put1);
943      assertFalse(ok);
944
945      // row doesn't exist, so using "ifNotExists" should be considered "match".
946      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER).ifNotExists().thenPut(put1);
947      assertTrue(ok);
948
949      // row now exists, so using "ifNotExists" should be considered "not match".
950      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER).ifNotExists().thenPut(put1);
951      assertFalse(ok);
952
953      Put put2 = new Put(ROW);
954      put2.addColumn(FAMILY, QUALIFIER, value2);
955
956      // row now exists, use the matching value to check
957      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER).ifEquals(VALUE).thenPut(put2);
958      assertTrue(ok);
959
960      Put put3 = new Put(anotherrow);
961      put3.addColumn(FAMILY, QUALIFIER, VALUE);
962
963      // try to do CheckAndPut on different rows
964      try {
965        table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER).ifEquals(value2).thenPut(put3);
966        fail("trying to check and modify different rows should have failed.");
967      } catch (Exception ignored) {
968      }
969    }
970  }
971
972  @Test
973  public void testCheckAndMutateWithTimeRange() throws IOException {
974    try (Table table = TEST_UTIL.createTable(name.getTableName(), FAMILY)) {
975      final long ts = EnvironmentEdgeManager.currentTime() / 2;
976      Put put = new Put(ROW);
977      put.addColumn(FAMILY, QUALIFIER, ts, VALUE);
978
979      boolean ok =
980        table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER).ifNotExists().thenPut(put);
981      assertTrue(ok);
982
983      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
984        .timeRange(TimeRange.at(ts + 10000)).ifEquals(VALUE).thenPut(put);
985      assertFalse(ok);
986
987      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
988        .timeRange(TimeRange.from(ts + 10000)).ifEquals(VALUE).thenPut(put);
989      assertFalse(ok);
990
991      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
992        .timeRange(TimeRange.between(ts + 10000, ts + 20000)).ifEquals(VALUE).thenPut(put);
993      assertFalse(ok);
994
995      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER).timeRange(TimeRange.until(ts))
996        .ifEquals(VALUE).thenPut(put);
997      assertFalse(ok);
998
999      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER).timeRange(TimeRange.at(ts))
1000        .ifEquals(VALUE).thenPut(put);
1001      assertTrue(ok);
1002
1003      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER).timeRange(TimeRange.from(ts))
1004        .ifEquals(VALUE).thenPut(put);
1005      assertTrue(ok);
1006
1007      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1008        .timeRange(TimeRange.between(ts, ts + 20000)).ifEquals(VALUE).thenPut(put);
1009      assertTrue(ok);
1010
1011      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1012        .timeRange(TimeRange.until(ts + 10000)).ifEquals(VALUE).thenPut(put);
1013      assertTrue(ok);
1014
1015      RowMutations rm = new RowMutations(ROW).add((Mutation) put);
1016      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1017        .timeRange(TimeRange.at(ts + 10000)).ifEquals(VALUE).thenMutate(rm);
1018      assertFalse(ok);
1019
1020      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER).timeRange(TimeRange.at(ts))
1021        .ifEquals(VALUE).thenMutate(rm);
1022      assertTrue(ok);
1023
1024      Delete delete = new Delete(ROW).addColumn(FAMILY, QUALIFIER);
1025
1026      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1027        .timeRange(TimeRange.at(ts + 10000)).ifEquals(VALUE).thenDelete(delete);
1028      assertFalse(ok);
1029
1030      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER).timeRange(TimeRange.at(ts))
1031        .ifEquals(VALUE).thenDelete(delete);
1032      assertTrue(ok);
1033    }
1034  }
1035
1036  @Test
1037  public void testCheckAndPutWithCompareOp() throws IOException {
1038    final byte[] value1 = Bytes.toBytes("aaaa");
1039    final byte[] value2 = Bytes.toBytes("bbbb");
1040    final byte[] value3 = Bytes.toBytes("cccc");
1041    final byte[] value4 = Bytes.toBytes("dddd");
1042
1043    try (Table table = TEST_UTIL.createTable(name.getTableName(), FAMILY)) {
1044
1045      Put put2 = new Put(ROW);
1046      put2.addColumn(FAMILY, QUALIFIER, value2);
1047
1048      Put put3 = new Put(ROW);
1049      put3.addColumn(FAMILY, QUALIFIER, value3);
1050
1051      // row doesn't exist, so using "ifNotExists" should be considered "match".
1052      boolean ok =
1053        table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER).ifNotExists().thenPut(put2);
1054      assertTrue(ok);
1055
1056      // cell = "bbbb", using "aaaa" to compare only LESS/LESS_OR_EQUAL/NOT_EQUAL
1057      // turns out "match"
1058      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1059        .ifMatches(CompareOperator.GREATER, value1).thenPut(put2);
1060      assertFalse(ok);
1061      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1062        .ifMatches(CompareOperator.EQUAL, value1).thenPut(put2);
1063      assertFalse(ok);
1064      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1065        .ifMatches(CompareOperator.GREATER_OR_EQUAL, value1).thenPut(put2);
1066      assertFalse(ok);
1067      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1068        .ifMatches(CompareOperator.LESS, value1).thenPut(put2);
1069      assertTrue(ok);
1070      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1071        .ifMatches(CompareOperator.LESS_OR_EQUAL, value1).thenPut(put2);
1072      assertTrue(ok);
1073      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1074        .ifMatches(CompareOperator.NOT_EQUAL, value1).thenPut(put3);
1075      assertTrue(ok);
1076
1077      // cell = "cccc", using "dddd" to compare only LARGER/LARGER_OR_EQUAL/NOT_EQUAL
1078      // turns out "match"
1079      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1080        .ifMatches(CompareOperator.LESS, value4).thenPut(put3);
1081      assertFalse(ok);
1082      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1083        .ifMatches(CompareOperator.LESS_OR_EQUAL, value4).thenPut(put3);
1084      assertFalse(ok);
1085      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1086        .ifMatches(CompareOperator.EQUAL, value4).thenPut(put3);
1087      assertFalse(ok);
1088      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1089        .ifMatches(CompareOperator.GREATER, value4).thenPut(put3);
1090      assertTrue(ok);
1091      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1092        .ifMatches(CompareOperator.GREATER_OR_EQUAL, value4).thenPut(put3);
1093      assertTrue(ok);
1094      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1095        .ifMatches(CompareOperator.NOT_EQUAL, value4).thenPut(put2);
1096      assertTrue(ok);
1097
1098      // cell = "bbbb", using "bbbb" to compare only GREATER_OR_EQUAL/LESS_OR_EQUAL/EQUAL
1099      // turns out "match"
1100      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1101        .ifMatches(CompareOperator.GREATER, value2).thenPut(put2);
1102      assertFalse(ok);
1103      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1104        .ifMatches(CompareOperator.NOT_EQUAL, value2).thenPut(put2);
1105      assertFalse(ok);
1106      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1107        .ifMatches(CompareOperator.LESS, value2).thenPut(put2);
1108      assertFalse(ok);
1109      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1110        .ifMatches(CompareOperator.GREATER_OR_EQUAL, value2).thenPut(put2);
1111      assertTrue(ok);
1112      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1113        .ifMatches(CompareOperator.LESS_OR_EQUAL, value2).thenPut(put2);
1114      assertTrue(ok);
1115      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1116        .ifMatches(CompareOperator.EQUAL, value2).thenPut(put3);
1117      assertTrue(ok);
1118    }
1119  }
1120
1121  @Test
1122  public void testCheckAndDelete() throws IOException {
1123    final byte[] value1 = Bytes.toBytes("aaaa");
1124
1125    try (Table table = TEST_UTIL.createTable(name.getTableName(), FAMILY)) {
1126
1127      Put put = new Put(ROW);
1128      put.addColumn(FAMILY, QUALIFIER, value1);
1129      table.put(put);
1130
1131      Delete delete = new Delete(ROW);
1132      delete.addColumns(FAMILY, QUALIFIER);
1133
1134      boolean ok =
1135        table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER).ifEquals(value1).thenDelete(delete);
1136      assertTrue(ok);
1137    }
1138  }
1139
1140  @Test
1141  public void testCheckAndDeleteWithCompareOp() throws IOException {
1142    final byte[] value1 = Bytes.toBytes("aaaa");
1143    final byte[] value2 = Bytes.toBytes("bbbb");
1144    final byte[] value3 = Bytes.toBytes("cccc");
1145    final byte[] value4 = Bytes.toBytes("dddd");
1146
1147    try (Table table = TEST_UTIL.createTable(name.getTableName(), FAMILY)) {
1148
1149      Put put2 = new Put(ROW);
1150      put2.addColumn(FAMILY, QUALIFIER, value2);
1151      table.put(put2);
1152
1153      Put put3 = new Put(ROW);
1154      put3.addColumn(FAMILY, QUALIFIER, value3);
1155
1156      Delete delete = new Delete(ROW);
1157      delete.addColumns(FAMILY, QUALIFIER);
1158
1159      // cell = "bbbb", using "aaaa" to compare only LESS/LESS_OR_EQUAL/NOT_EQUAL
1160      // turns out "match"
1161      boolean ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1162        .ifMatches(CompareOperator.GREATER, value1).thenDelete(delete);
1163      assertFalse(ok);
1164      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1165        .ifMatches(CompareOperator.EQUAL, value1).thenDelete(delete);
1166      assertFalse(ok);
1167      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1168        .ifMatches(CompareOperator.GREATER_OR_EQUAL, value1).thenDelete(delete);
1169      assertFalse(ok);
1170      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1171        .ifMatches(CompareOperator.LESS, value1).thenDelete(delete);
1172      assertTrue(ok);
1173      table.put(put2);
1174      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1175        .ifMatches(CompareOperator.LESS_OR_EQUAL, value1).thenDelete(delete);
1176      assertTrue(ok);
1177      table.put(put2);
1178      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1179        .ifMatches(CompareOperator.NOT_EQUAL, value1).thenDelete(delete);
1180      assertTrue(ok);
1181
1182      // cell = "cccc", using "dddd" to compare only LARGER/LARGER_OR_EQUAL/NOT_EQUAL
1183      // turns out "match"
1184      table.put(put3);
1185      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1186        .ifMatches(CompareOperator.LESS, value4).thenDelete(delete);
1187      assertFalse(ok);
1188      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1189        .ifMatches(CompareOperator.LESS_OR_EQUAL, value4).thenDelete(delete);
1190      assertFalse(ok);
1191      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1192        .ifMatches(CompareOperator.EQUAL, value4).thenDelete(delete);
1193      assertFalse(ok);
1194      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1195        .ifMatches(CompareOperator.GREATER, value4).thenDelete(delete);
1196      assertTrue(ok);
1197      table.put(put3);
1198      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1199        .ifMatches(CompareOperator.GREATER_OR_EQUAL, value4).thenDelete(delete);
1200      assertTrue(ok);
1201      table.put(put3);
1202      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1203        .ifMatches(CompareOperator.NOT_EQUAL, value4).thenDelete(delete);
1204      assertTrue(ok);
1205
1206      // cell = "bbbb", using "bbbb" to compare only GREATER_OR_EQUAL/LESS_OR_EQUAL/EQUAL
1207      // turns out "match"
1208      table.put(put2);
1209      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1210        .ifMatches(CompareOperator.GREATER, value2).thenDelete(delete);
1211      assertFalse(ok);
1212      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1213        .ifMatches(CompareOperator.NOT_EQUAL, value2).thenDelete(delete);
1214      assertFalse(ok);
1215      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1216        .ifMatches(CompareOperator.LESS, value2).thenDelete(delete);
1217      assertFalse(ok);
1218      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1219        .ifMatches(CompareOperator.GREATER_OR_EQUAL, value2).thenDelete(delete);
1220      assertTrue(ok);
1221      table.put(put2);
1222      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1223        .ifMatches(CompareOperator.LESS_OR_EQUAL, value2).thenDelete(delete);
1224      assertTrue(ok);
1225      table.put(put2);
1226      ok = table.checkAndMutate(ROW, FAMILY).qualifier(QUALIFIER)
1227        .ifMatches(CompareOperator.EQUAL, value2).thenDelete(delete);
1228      assertTrue(ok);
1229    }
1230  }
1231
1232  /**
1233   * Test ScanMetrics
1234   */
1235  @Test
1236  @SuppressWarnings({ "unused", "checkstyle:EmptyBlock" })
1237  public void testScanMetrics() throws Exception {
1238    final TableName tableName = name.getTableName();
1239
1240    // Set up test table:
1241    // Create table:
1242    try (Table ht = TEST_UTIL.createMultiRegionTable(tableName, FAMILY)) {
1243      int numOfRegions;
1244      try (RegionLocator r = TEST_UTIL.getConnection().getRegionLocator(tableName)) {
1245        numOfRegions = r.getStartKeys().length;
1246      }
1247      // Create 3 rows in the table, with rowkeys starting with "zzz*" so that
1248      // scan are forced to hit all the regions.
1249      Put put1 = new Put(Bytes.toBytes("zzz1"));
1250      put1.addColumn(FAMILY, QUALIFIER, VALUE);
1251      Put put2 = new Put(Bytes.toBytes("zzz2"));
1252      put2.addColumn(FAMILY, QUALIFIER, VALUE);
1253      Put put3 = new Put(Bytes.toBytes("zzz3"));
1254      put3.addColumn(FAMILY, QUALIFIER, VALUE);
1255      ht.put(Arrays.asList(put1, put2, put3));
1256
1257      Scan scan1 = new Scan();
1258      int numRecords = 0;
1259      try (ResultScanner scanner = ht.getScanner(scan1)) {
1260        for (Result result : scanner) {
1261          numRecords++;
1262        }
1263
1264        LOG.info("test data has {} records.", numRecords);
1265
1266        // by default, scan metrics collection is turned off
1267        assertNull(scanner.getScanMetrics());
1268      }
1269
1270      // turn on scan metrics
1271      Scan scan2 = new Scan();
1272      scan2.setScanMetricsEnabled(true);
1273      scan2.setCaching(numRecords + 1);
1274      try (ResultScanner scanner = ht.getScanner(scan2)) {
1275        for (Result result : scanner.next(numRecords - 1)) {
1276        }
1277        assertNotNull(scanner.getScanMetrics());
1278      }
1279
1280      // set caching to 1, because metrics are collected in each roundtrip only
1281      scan2 = new Scan();
1282      scan2.setScanMetricsEnabled(true);
1283      scan2.setCaching(1);
1284      try (ResultScanner scanner = ht.getScanner(scan2)) {
1285        // per HBASE-5717, this should still collect even if you don't run all the way to
1286        // the end of the scanner. So this is asking for 2 of the 3 rows we inserted.
1287        for (Result result : scanner.next(numRecords - 1)) {
1288        }
1289        ScanMetrics scanMetrics = scanner.getScanMetrics();
1290        assertEquals("Did not access all the regions in the table", numOfRegions,
1291          scanMetrics.countOfRegions.get());
1292      }
1293
1294      // check byte counters
1295      scan2 = new Scan();
1296      scan2.setScanMetricsEnabled(true);
1297      scan2.setCaching(1);
1298      try (ResultScanner scanner = ht.getScanner(scan2)) {
1299        int numBytes = 0;
1300        for (Result result : scanner) {
1301          for (Cell cell : result.listCells()) {
1302            numBytes += PrivateCellUtil.estimatedSerializedSizeOf(cell);
1303          }
1304        }
1305        ScanMetrics scanMetrics = scanner.getScanMetrics();
1306        assertEquals("Did not count the result bytes", numBytes,
1307          scanMetrics.countOfBytesInResults.get());
1308      }
1309
1310      // check byte counters on a small scan
1311      scan2 = new Scan();
1312      scan2.setScanMetricsEnabled(true);
1313      scan2.setCaching(1);
1314      scan2.setSmall(true);
1315      try (ResultScanner scanner = ht.getScanner(scan2)) {
1316        int numBytes = 0;
1317        for (Result result : scanner) {
1318          for (Cell cell : result.listCells()) {
1319            numBytes += PrivateCellUtil.estimatedSerializedSizeOf(cell);
1320          }
1321        }
1322        ScanMetrics scanMetrics = scanner.getScanMetrics();
1323        assertEquals("Did not count the result bytes", numBytes,
1324          scanMetrics.countOfBytesInResults.get());
1325      }
1326
1327      // now, test that the metrics are still collected even if you don't call close, but do
1328      // run past the end of all the records
1329      /**
1330       * There seems to be a timing issue here. Comment out for now. Fix when time. Scan
1331       * scanWithoutClose = new Scan(); scanWithoutClose.setCaching(1);
1332       * scanWithoutClose.setScanMetricsEnabled(true); ResultScanner scannerWithoutClose =
1333       * ht.getScanner(scanWithoutClose); for (Result result : scannerWithoutClose.next(numRecords +
1334       * 1)) { } ScanMetrics scanMetricsWithoutClose = getScanMetrics(scanWithoutClose);
1335       * assertEquals("Did not access all the regions in the table", numOfRegions,
1336       * scanMetricsWithoutClose.countOfRegions.get());
1337       */
1338
1339      // finally,
1340      // test that the metrics are collected correctly if you both run past all the records,
1341      // AND close the scanner
1342      Scan scanWithClose = new Scan();
1343      // make sure we can set caching up to the number of a scanned values
1344      scanWithClose.setCaching(numRecords);
1345      scanWithClose.setScanMetricsEnabled(true);
1346      try (ResultScanner scannerWithClose = ht.getScanner(scanWithClose)) {
1347        for (Result result : scannerWithClose.next(numRecords + 1)) {
1348        }
1349        scannerWithClose.close();
1350        ScanMetrics scanMetricsWithClose = scannerWithClose.getScanMetrics();
1351        assertEquals("Did not access all the regions in the table", numOfRegions,
1352          scanMetricsWithClose.countOfRegions.get());
1353      }
1354    } finally {
1355      TEST_UTIL.deleteTable(tableName);
1356    }
1357  }
1358
1359  /**
1360   * Tests that cache on write works all the way up from the client-side. Performs inserts, flushes,
1361   * and compactions, verifying changes in the block cache along the way.
1362   */
1363  @Test
1364  public void testCacheOnWriteEvictOnClose() throws Exception {
1365    final TableName tableName = name.getTableName();
1366    byte[] data = Bytes.toBytes("data");
1367    try (Table table = TEST_UTIL.createTable(tableName, FAMILY)) {
1368      try (RegionLocator locator = TEST_UTIL.getConnection().getRegionLocator(tableName)) {
1369        // get the block cache and region
1370        String regionName = locator.getAllRegionLocations().get(0).getRegion().getEncodedName();
1371
1372        HRegion region = TEST_UTIL.getRSForFirstRegionInTable(tableName).getRegion(regionName);
1373        HStore store = region.getStores().iterator().next();
1374        CacheConfig cacheConf = store.getCacheConfig();
1375        cacheConf.setCacheDataOnWrite(true);
1376        cacheConf.setEvictOnClose(true);
1377        BlockCache cache = cacheConf.getBlockCache().get();
1378
1379        // establish baseline stats
1380        long startBlockCount = cache.getBlockCount();
1381        long startBlockHits = cache.getStats().getHitCount();
1382        long startBlockMiss = cache.getStats().getMissCount();
1383
1384        // wait till baseline is stable, (minimal 500 ms)
1385        for (int i = 0; i < 5; i++) {
1386          Thread.sleep(100);
1387          if (
1388            startBlockCount != cache.getBlockCount()
1389              || startBlockHits != cache.getStats().getHitCount()
1390              || startBlockMiss != cache.getStats().getMissCount()
1391          ) {
1392            startBlockCount = cache.getBlockCount();
1393            startBlockHits = cache.getStats().getHitCount();
1394            startBlockMiss = cache.getStats().getMissCount();
1395            i = -1;
1396          }
1397        }
1398
1399        // insert data
1400        Put put = new Put(ROW);
1401        put.addColumn(FAMILY, QUALIFIER, data);
1402        table.put(put);
1403        assertTrue(Bytes.equals(table.get(new Get(ROW)).value(), data));
1404
1405        // data was in memstore so don't expect any changes
1406        assertEquals(startBlockCount, cache.getBlockCount());
1407        assertEquals(startBlockHits, cache.getStats().getHitCount());
1408        assertEquals(startBlockMiss, cache.getStats().getMissCount());
1409
1410        // flush the data
1411        LOG.debug("Flushing cache");
1412        region.flush(true);
1413
1414        // expect two more blocks in cache - DATA and ROOT_INDEX
1415        // , no change in hits/misses
1416        long expectedBlockCount = startBlockCount + 2;
1417        long expectedBlockHits = startBlockHits;
1418        long expectedBlockMiss = startBlockMiss;
1419        assertEquals(expectedBlockCount, cache.getBlockCount());
1420        assertEquals(expectedBlockHits, cache.getStats().getHitCount());
1421        assertEquals(expectedBlockMiss, cache.getStats().getMissCount());
1422        // read the data and expect same blocks, one new hit, no misses
1423        assertTrue(Bytes.equals(table.get(new Get(ROW)).value(), data));
1424        assertEquals(expectedBlockCount, cache.getBlockCount());
1425        assertEquals(++expectedBlockHits, cache.getStats().getHitCount());
1426        assertEquals(expectedBlockMiss, cache.getStats().getMissCount());
1427        // insert a second column, read the row, no new blocks, one new hit
1428        byte[] QUALIFIER2 = Bytes.add(QUALIFIER, QUALIFIER);
1429        byte[] data2 = Bytes.add(data, data);
1430        put = new Put(ROW);
1431        put.addColumn(FAMILY, QUALIFIER2, data2);
1432        table.put(put);
1433        Result r = table.get(new Get(ROW));
1434        assertTrue(Bytes.equals(r.getValue(FAMILY, QUALIFIER), data));
1435        assertTrue(Bytes.equals(r.getValue(FAMILY, QUALIFIER2), data2));
1436        assertEquals(expectedBlockCount, cache.getBlockCount());
1437        assertEquals(++expectedBlockHits, cache.getStats().getHitCount());
1438        assertEquals(expectedBlockMiss, cache.getStats().getMissCount());
1439        // flush, one new block
1440        System.out.println("Flushing cache");
1441        region.flush(true);
1442
1443        // + 1 for Index Block, +1 for data block
1444        expectedBlockCount += 2;
1445        assertEquals(expectedBlockCount, cache.getBlockCount());
1446        assertEquals(expectedBlockHits, cache.getStats().getHitCount());
1447        assertEquals(expectedBlockMiss, cache.getStats().getMissCount());
1448        // compact, net minus two blocks, two hits, no misses
1449        System.out.println("Compacting");
1450        assertEquals(2, store.getStorefilesCount());
1451        store.triggerMajorCompaction();
1452        region.compact(true);
1453        store.closeAndArchiveCompactedFiles();
1454        waitForStoreFileCount(store, 1, 10000); // wait 10 seconds max
1455        assertEquals(1, store.getStorefilesCount());
1456        // evicted two data blocks and two index blocks and compaction does not cache new blocks
1457        expectedBlockCount = 0;
1458        assertEquals(expectedBlockCount, cache.getBlockCount());
1459        expectedBlockHits += 2;
1460        assertEquals(expectedBlockMiss, cache.getStats().getMissCount());
1461        assertEquals(expectedBlockHits, cache.getStats().getHitCount());
1462        // read the row, this should be a cache miss because we don't cache data
1463        // blocks on compaction
1464        r = table.get(new Get(ROW));
1465        assertTrue(Bytes.equals(r.getValue(FAMILY, QUALIFIER), data));
1466        assertTrue(Bytes.equals(r.getValue(FAMILY, QUALIFIER2), data2));
1467        expectedBlockCount += 1; // cached one data block
1468        assertEquals(expectedBlockCount, cache.getBlockCount());
1469        assertEquals(expectedBlockHits, cache.getStats().getHitCount());
1470        assertEquals(++expectedBlockMiss, cache.getStats().getMissCount());
1471      }
1472    }
1473  }
1474
1475  private void waitForStoreFileCount(HStore store, int count, int timeout)
1476    throws InterruptedException {
1477    long start = EnvironmentEdgeManager.currentTime();
1478    while (
1479      start + timeout > EnvironmentEdgeManager.currentTime() && store.getStorefilesCount() != count
1480    ) {
1481      Thread.sleep(100);
1482    }
1483    System.out.println("start=" + start + ", now=" + EnvironmentEdgeManager.currentTime() + ", cur="
1484      + store.getStorefilesCount());
1485    assertEquals(count, store.getStorefilesCount());
1486  }
1487
1488  /**
1489   * Tests the non cached version of getRegionLocator by moving a region.
1490   */
1491  @Test
1492  public void testNonCachedGetRegionLocation() throws Exception {
1493    // Test Initialization.
1494    final TableName tableName = name.getTableName();
1495    byte[] family1 = Bytes.toBytes("f1");
1496    byte[] family2 = Bytes.toBytes("f2");
1497    try (Table ignored = TEST_UTIL.createTable(tableName, new byte[][] { family1, family2 }, 10);
1498      Admin admin = TEST_UTIL.getAdmin();
1499      RegionLocator locator = TEST_UTIL.getConnection().getRegionLocator(tableName)) {
1500      List<HRegionLocation> allRegionLocations = locator.getAllRegionLocations();
1501      assertEquals(1, allRegionLocations.size());
1502      RegionInfo regionInfo = allRegionLocations.get(0).getRegion();
1503      ServerName addrBefore = allRegionLocations.get(0).getServerName();
1504      // Verify region location before move.
1505      HRegionLocation addrCache = locator.getRegionLocation(regionInfo.getStartKey(), false);
1506      HRegionLocation addrNoCache = locator.getRegionLocation(regionInfo.getStartKey(), true);
1507
1508      assertEquals(addrBefore.getPort(), addrCache.getPort());
1509      assertEquals(addrBefore.getPort(), addrNoCache.getPort());
1510
1511      // Make sure more than one server.
1512      if (TEST_UTIL.getMiniHBaseCluster().getLiveRegionServerThreads().size() <= 1) {
1513        TEST_UTIL.getMiniHBaseCluster().startRegionServer();
1514        Waiter.waitFor(TEST_UTIL.getConfiguration(), 30000, new Waiter.Predicate<Exception>() {
1515          @Override
1516          public boolean evaluate() throws Exception {
1517            return TEST_UTIL.getMiniHBaseCluster().getLiveRegionServerThreads().size() > 1;
1518          }
1519        });
1520      }
1521
1522      ServerName addrAfter = null;
1523      // Now move the region to a different server.
1524      for (int i = 0; i
1525          < TEST_UTIL.getMiniHBaseCluster().getLiveRegionServerThreads().size(); i++) {
1526        HRegionServer regionServer = TEST_UTIL.getHBaseCluster().getRegionServer(i);
1527        ServerName addr = regionServer.getServerName();
1528        if (addr.getPort() != addrBefore.getPort()) {
1529          admin.move(regionInfo.getEncodedNameAsBytes(), addr);
1530          // Wait for the region to move.
1531          Thread.sleep(5000);
1532          addrAfter = addr;
1533          break;
1534        }
1535      }
1536
1537      // Verify the region was moved.
1538      addrCache = locator.getRegionLocation(regionInfo.getStartKey(), false);
1539      addrNoCache = locator.getRegionLocation(regionInfo.getStartKey(), true);
1540      assertNotNull(addrAfter);
1541      assertTrue(addrAfter.getPort() != addrCache.getPort());
1542      assertEquals(addrAfter.getPort(), addrNoCache.getPort());
1543    }
1544  }
1545
1546  /**
1547   * Tests getRegionsInRange by creating some regions over which a range of keys spans; then
1548   * changing the key range.
1549   */
1550  @Test
1551  public void testGetRegionsInRange() throws Exception {
1552    // Test Initialization.
1553    byte[] startKey = Bytes.toBytes("ddc");
1554    byte[] endKey = Bytes.toBytes("mmm");
1555    TableName tableName = name.getTableName();
1556    TEST_UTIL.createMultiRegionTable(tableName, new byte[][] { FAMILY }, 10);
1557
1558    int numOfRegions;
1559    try (RegionLocator r = TEST_UTIL.getConnection().getRegionLocator(tableName)) {
1560      numOfRegions = r.getStartKeys().length;
1561    }
1562    assertEquals(26, numOfRegions);
1563
1564    // Get the regions in this range
1565    List<HRegionLocation> regionsList = getRegionsInRange(tableName, startKey, endKey);
1566    assertEquals(10, regionsList.size());
1567
1568    // Change the start key
1569    startKey = Bytes.toBytes("fff");
1570    regionsList = getRegionsInRange(tableName, startKey, endKey);
1571    assertEquals(7, regionsList.size());
1572
1573    // Change the end key
1574    endKey = Bytes.toBytes("nnn");
1575    regionsList = getRegionsInRange(tableName, startKey, endKey);
1576    assertEquals(8, regionsList.size());
1577
1578    // Empty start key
1579    regionsList = getRegionsInRange(tableName, HConstants.EMPTY_START_ROW, endKey);
1580    assertEquals(13, regionsList.size());
1581
1582    // Empty end key
1583    regionsList = getRegionsInRange(tableName, startKey, HConstants.EMPTY_END_ROW);
1584    assertEquals(21, regionsList.size());
1585
1586    // Both start and end keys empty
1587    regionsList =
1588      getRegionsInRange(tableName, HConstants.EMPTY_START_ROW, HConstants.EMPTY_END_ROW);
1589    assertEquals(26, regionsList.size());
1590
1591    // Change the end key to somewhere in the last block
1592    endKey = Bytes.toBytes("zzz1");
1593    regionsList = getRegionsInRange(tableName, startKey, endKey);
1594    assertEquals(21, regionsList.size());
1595
1596    // Change the start key to somewhere in the first block
1597    startKey = Bytes.toBytes("aac");
1598    regionsList = getRegionsInRange(tableName, startKey, endKey);
1599    assertEquals(26, regionsList.size());
1600
1601    // Make start and end key the same
1602    startKey = Bytes.toBytes("ccc");
1603    endKey = Bytes.toBytes("ccc");
1604    regionsList = getRegionsInRange(tableName, startKey, endKey);
1605    assertEquals(1, regionsList.size());
1606  }
1607
1608  private List<HRegionLocation> getRegionsInRange(TableName tableName, byte[] startKey,
1609    byte[] endKey) throws IOException {
1610    List<HRegionLocation> regionsInRange = new ArrayList<>();
1611    byte[] currentKey = startKey;
1612    final boolean endKeyIsEndOfTable = Bytes.equals(endKey, HConstants.EMPTY_END_ROW);
1613    try (RegionLocator r = TEST_UTIL.getConnection().getRegionLocator(tableName)) {
1614      do {
1615        HRegionLocation regionLocation = r.getRegionLocation(currentKey);
1616        regionsInRange.add(regionLocation);
1617        currentKey = regionLocation.getRegion().getEndKey();
1618      } while (
1619        !Bytes.equals(currentKey, HConstants.EMPTY_END_ROW)
1620          && (endKeyIsEndOfTable || Bytes.compareTo(currentKey, endKey) < 0)
1621      );
1622      return regionsInRange;
1623    }
1624  }
1625
1626  @Test
1627  public void testJira6912() throws Exception {
1628    final TableName tableName = name.getTableName();
1629    try (Table foo = TEST_UTIL.createTable(tableName, new byte[][] { FAMILY }, 10)) {
1630
1631      List<Put> puts = new ArrayList<>();
1632      for (int i = 0; i != 100; i++) {
1633        Put put = new Put(Bytes.toBytes(i));
1634        put.addColumn(FAMILY, FAMILY, Bytes.toBytes(i));
1635        puts.add(put);
1636      }
1637      foo.put(puts);
1638      // If i comment this out it works
1639      TEST_UTIL.flush();
1640
1641      Scan scan = new Scan();
1642      scan.setStartRow(Bytes.toBytes(1));
1643      scan.setStopRow(Bytes.toBytes(3));
1644      scan.addColumn(FAMILY, FAMILY);
1645      scan.setFilter(
1646        new RowFilter(CompareOperator.NOT_EQUAL, new BinaryComparator(Bytes.toBytes(1))));
1647
1648      try (ResultScanner scanner = foo.getScanner(scan)) {
1649        Result[] bar = scanner.next(100);
1650        assertEquals(1, bar.length);
1651      }
1652    }
1653  }
1654
1655  @Test
1656  public void testScan_NullQualifier() throws IOException {
1657    try (Table table = TEST_UTIL.createTable(name.getTableName(), FAMILY)) {
1658      Put put = new Put(ROW);
1659      put.addColumn(FAMILY, QUALIFIER, VALUE);
1660      table.put(put);
1661
1662      put = new Put(ROW);
1663      put.addColumn(FAMILY, null, VALUE);
1664      table.put(put);
1665      LOG.info("Row put");
1666
1667      Scan scan = new Scan();
1668      scan.addColumn(FAMILY, null);
1669
1670      ResultScanner scanner = table.getScanner(scan);
1671      Result[] bar = scanner.next(100);
1672      assertEquals(1, bar.length);
1673      assertEquals(1, bar[0].size());
1674
1675      scan = new Scan();
1676      scan.addFamily(FAMILY);
1677
1678      scanner = table.getScanner(scan);
1679      bar = scanner.next(100);
1680      assertEquals(1, bar.length);
1681      assertEquals(2, bar[0].size());
1682    }
1683  }
1684
1685  @Test
1686  public void testNegativeTimestamp() throws IOException {
1687    try (Table table = TEST_UTIL.createTable(name.getTableName(), FAMILY)) {
1688
1689      try {
1690        Put put = new Put(ROW, -1);
1691        put.addColumn(FAMILY, QUALIFIER, VALUE);
1692        table.put(put);
1693        fail("Negative timestamps should not have been allowed");
1694      } catch (IllegalArgumentException ex) {
1695        assertTrue(ex.getMessage().contains("negative"));
1696      }
1697
1698      try {
1699        Put put = new Put(ROW);
1700        long ts = -1;
1701        put.addColumn(FAMILY, QUALIFIER, ts, VALUE);
1702        table.put(put);
1703        fail("Negative timestamps should not have been allowed");
1704      } catch (IllegalArgumentException ex) {
1705        assertTrue(ex.getMessage().contains("negative"));
1706      }
1707
1708      try {
1709        Delete delete = new Delete(ROW, -1);
1710        table.delete(delete);
1711        fail("Negative timestamps should not have been allowed");
1712      } catch (IllegalArgumentException ex) {
1713        assertTrue(ex.getMessage().contains("negative"));
1714      }
1715
1716      try {
1717        Delete delete = new Delete(ROW);
1718        delete.addFamily(FAMILY, -1);
1719        table.delete(delete);
1720        fail("Negative timestamps should not have been allowed");
1721      } catch (IllegalArgumentException ex) {
1722        assertTrue(ex.getMessage().contains("negative"));
1723      }
1724
1725      try {
1726        Scan scan = new Scan();
1727        scan.setTimeRange(-1, 1);
1728        table.getScanner(scan);
1729        fail("Negative timestamps should not have been allowed");
1730      } catch (IllegalArgumentException ex) {
1731        assertTrue(ex.getMessage().contains("negative"));
1732      }
1733
1734      // KeyValue should allow negative timestamps for backwards compat. Otherwise, if the user
1735      // already has negative timestamps in cluster data, HBase won't be able to handle that
1736      try {
1737        new KeyValue(Bytes.toBytes(42), Bytes.toBytes(42), Bytes.toBytes(42), -1,
1738          Bytes.toBytes(42));
1739      } catch (IllegalArgumentException ex) {
1740        fail("KeyValue SHOULD allow negative timestamps");
1741      }
1742
1743    }
1744  }
1745
1746  @Test
1747  public void testRawScanRespectsVersions() throws Exception {
1748    final TableName tableName = name.getTableName();
1749    try (Table table = TEST_UTIL.createTable(tableName, FAMILY)) {
1750      byte[] row = Bytes.toBytes("row");
1751
1752      // put the same row 4 times, with different values
1753      Put p = new Put(row);
1754      p.addColumn(FAMILY, QUALIFIER, 10, VALUE);
1755      table.put(p);
1756      p = new Put(row);
1757      p.addColumn(FAMILY, QUALIFIER, 11, ArrayUtils.add(VALUE, (byte) 2));
1758      table.put(p);
1759
1760      p = new Put(row);
1761      p.addColumn(FAMILY, QUALIFIER, 12, ArrayUtils.add(VALUE, (byte) 3));
1762      table.put(p);
1763
1764      p = new Put(row);
1765      p.addColumn(FAMILY, QUALIFIER, 13, ArrayUtils.add(VALUE, (byte) 4));
1766      table.put(p);
1767
1768      int versions = 4;
1769      Scan s = new Scan(row);
1770      // get all the possible versions
1771      s.setMaxVersions();
1772      s.setRaw(true);
1773
1774      try (ResultScanner scanner = table.getScanner(s)) {
1775        int count = 0;
1776        for (Result r : scanner) {
1777          assertEquals("Found an unexpected number of results for the row!", versions,
1778            r.listCells().size());
1779          count++;
1780        }
1781        assertEquals("Found more than a single row when raw scanning the table with a single row!",
1782          1, count);
1783      }
1784
1785      // then if we decrease the number of versions, but keep the scan raw, we should see exactly
1786      // that number of versions
1787      versions = 2;
1788      s.setMaxVersions(versions);
1789      try (ResultScanner scanner = table.getScanner(s)) {
1790        int count = 0;
1791        for (Result r : scanner) {
1792          assertEquals("Found an unexpected number of results for the row!", versions,
1793            r.listCells().size());
1794          count++;
1795        }
1796        assertEquals("Found more than a single row when raw scanning the table with a single row!",
1797          1, count);
1798      }
1799
1800      // finally, if we turn off raw scanning, but max out the number of versions, we should go back
1801      // to seeing just three
1802      versions = 3;
1803      s.setMaxVersions(versions);
1804      try (ResultScanner scanner = table.getScanner(s)) {
1805        int count = 0;
1806        for (Result r : scanner) {
1807          assertEquals("Found an unexpected number of results for the row!", versions,
1808            r.listCells().size());
1809          count++;
1810        }
1811        assertEquals("Found more than a single row when raw scanning the table with a single row!",
1812          1, count);
1813      }
1814
1815    }
1816    TEST_UTIL.deleteTable(tableName);
1817  }
1818
1819  @Test
1820  public void testEmptyFilterList() throws Exception {
1821    // Test Initialization.
1822    final TableName tableName = name.getTableName();
1823    try (Table table = TEST_UTIL.createTable(tableName, FAMILY)) {
1824
1825      // Insert one row each region
1826      Put put = new Put(Bytes.toBytes("row"));
1827      put.addColumn(FAMILY, QUALIFIER, VALUE);
1828      table.put(put);
1829
1830      List<Result> scanResults = new LinkedList<>();
1831      Scan scan = new Scan();
1832      scan.setFilter(new FilterList());
1833      try (ResultScanner scanner = table.getScanner(scan)) {
1834        for (Result r : scanner) {
1835          scanResults.add(r);
1836        }
1837      }
1838      assertEquals(1, scanResults.size());
1839      Get g = new Get(Bytes.toBytes("row"));
1840      g.setFilter(new FilterList());
1841      Result getResult = table.get(g);
1842      Result scanResult = scanResults.get(0);
1843      assertEquals(scanResult.rawCells().length, getResult.rawCells().length);
1844      for (int i = 0; i != scanResult.rawCells().length; ++i) {
1845        Cell scanCell = scanResult.rawCells()[i];
1846        Cell getCell = getResult.rawCells()[i];
1847        assertEquals(0, Bytes.compareTo(CellUtil.cloneRow(scanCell), CellUtil.cloneRow(getCell)));
1848        assertEquals(0,
1849          Bytes.compareTo(CellUtil.cloneFamily(scanCell), CellUtil.cloneFamily(getCell)));
1850        assertEquals(0,
1851          Bytes.compareTo(CellUtil.cloneQualifier(scanCell), CellUtil.cloneQualifier(getCell)));
1852        assertEquals(0,
1853          Bytes.compareTo(CellUtil.cloneValue(scanCell), CellUtil.cloneValue(getCell)));
1854      }
1855    }
1856  }
1857
1858  @Test
1859  public void testSmallScan() throws Exception {
1860    // Test Initialization.
1861    final TableName tableName = name.getTableName();
1862    try (Table table = TEST_UTIL.createTable(tableName, FAMILY)) {
1863
1864      // Insert one row each region
1865      int insertNum = 10;
1866      for (int i = 0; i < 10; i++) {
1867        Put put = new Put(Bytes.toBytes("row" + String.format("%03d", i)));
1868        put.addColumn(FAMILY, QUALIFIER, VALUE);
1869        table.put(put);
1870      }
1871
1872      // normal scan
1873      try (ResultScanner scanner = table.getScanner(new Scan())) {
1874        int count = 0;
1875        for (Result r : scanner) {
1876          assertFalse(r.isEmpty());
1877          count++;
1878        }
1879        assertEquals(insertNum, count);
1880      }
1881
1882      // small scan
1883      Scan scan = new Scan(HConstants.EMPTY_START_ROW, HConstants.EMPTY_END_ROW);
1884      scan.setSmall(true);
1885      scan.setCaching(2);
1886      try (ResultScanner scanner = table.getScanner(scan)) {
1887        int count = 0;
1888        for (Result r : scanner) {
1889          assertFalse(r.isEmpty());
1890          count++;
1891        }
1892        assertEquals(insertNum, count);
1893      }
1894    }
1895  }
1896
1897  @Test
1898  public void testSuperSimpleWithReverseScan() throws Exception {
1899    final TableName tableName = name.getTableName();
1900    try (Table ht = TEST_UTIL.createTable(tableName, FAMILY)) {
1901      Put put = new Put(Bytes.toBytes("0-b11111-0000000000000000000"));
1902      put.addColumn(FAMILY, QUALIFIER, VALUE);
1903      ht.put(put);
1904      put = new Put(Bytes.toBytes("0-b11111-0000000000000000002"));
1905      put.addColumn(FAMILY, QUALIFIER, VALUE);
1906      ht.put(put);
1907      put = new Put(Bytes.toBytes("0-b11111-0000000000000000004"));
1908      put.addColumn(FAMILY, QUALIFIER, VALUE);
1909      ht.put(put);
1910      put = new Put(Bytes.toBytes("0-b11111-0000000000000000006"));
1911      put.addColumn(FAMILY, QUALIFIER, VALUE);
1912      ht.put(put);
1913      put = new Put(Bytes.toBytes("0-b11111-0000000000000000008"));
1914      put.addColumn(FAMILY, QUALIFIER, VALUE);
1915      ht.put(put);
1916      put = new Put(Bytes.toBytes("0-b22222-0000000000000000001"));
1917      put.addColumn(FAMILY, QUALIFIER, VALUE);
1918      ht.put(put);
1919      put = new Put(Bytes.toBytes("0-b22222-0000000000000000003"));
1920      put.addColumn(FAMILY, QUALIFIER, VALUE);
1921      ht.put(put);
1922      put = new Put(Bytes.toBytes("0-b22222-0000000000000000005"));
1923      put.addColumn(FAMILY, QUALIFIER, VALUE);
1924      ht.put(put);
1925      put = new Put(Bytes.toBytes("0-b22222-0000000000000000007"));
1926      put.addColumn(FAMILY, QUALIFIER, VALUE);
1927      ht.put(put);
1928      put = new Put(Bytes.toBytes("0-b22222-0000000000000000009"));
1929      put.addColumn(FAMILY, QUALIFIER, VALUE);
1930      ht.put(put);
1931      Scan scan = new Scan(Bytes.toBytes("0-b11111-9223372036854775807"),
1932        Bytes.toBytes("0-b11111-0000000000000000000"));
1933      scan.setReversed(true);
1934      try (ResultScanner scanner = ht.getScanner(scan)) {
1935        Result result = scanner.next();
1936        assertTrue(Bytes.equals(result.getRow(), Bytes.toBytes("0-b11111-0000000000000000008")));
1937      }
1938    }
1939  }
1940
1941  @Test
1942  public void testFiltersWithReverseScan() throws Exception {
1943    final TableName tableName = name.getTableName();
1944    try (Table ht = TEST_UTIL.createTable(tableName, FAMILY)) {
1945      byte[][] ROWS = makeN(ROW, 10);
1946      byte[][] QUALIFIERS =
1947        { Bytes.toBytes("col0-<d2v1>-<d3v2>"), Bytes.toBytes("col1-<d2v1>-<d3v2>"),
1948          Bytes.toBytes("col2-<d2v1>-<d3v2>"), Bytes.toBytes("col3-<d2v1>-<d3v2>"),
1949          Bytes.toBytes("col4-<d2v1>-<d3v2>"), Bytes.toBytes("col5-<d2v1>-<d3v2>"),
1950          Bytes.toBytes("col6-<d2v1>-<d3v2>"), Bytes.toBytes("col7-<d2v1>-<d3v2>"),
1951          Bytes.toBytes("col8-<d2v1>-<d3v2>"), Bytes.toBytes("col9-<d2v1>-<d3v2>") };
1952      for (int i = 0; i < 10; i++) {
1953        Put put = new Put(ROWS[i]);
1954        put.addColumn(FAMILY, QUALIFIERS[i], VALUE);
1955        ht.put(put);
1956      }
1957      Scan scan = new Scan();
1958      scan.setReversed(true);
1959      scan.addFamily(FAMILY);
1960      Filter filter =
1961        new QualifierFilter(CompareOperator.EQUAL, new RegexStringComparator("col[1-5]"));
1962      scan.setFilter(filter);
1963      try (ResultScanner scanner = ht.getScanner(scan)) {
1964        int expectedIndex = 5;
1965        for (Result result : scanner) {
1966          assertEquals(1, result.size());
1967          Cell c = result.rawCells()[0];
1968          assertTrue(Bytes.equals(c.getRowArray(), c.getRowOffset(), c.getRowLength(),
1969            ROWS[expectedIndex], 0, ROWS[expectedIndex].length));
1970          assertTrue(
1971            Bytes.equals(c.getQualifierArray(), c.getQualifierOffset(), c.getQualifierLength(),
1972              QUALIFIERS[expectedIndex], 0, QUALIFIERS[expectedIndex].length));
1973          expectedIndex--;
1974        }
1975        assertEquals(0, expectedIndex);
1976      }
1977    }
1978  }
1979
1980  @Test
1981  public void testKeyOnlyFilterWithReverseScan() throws Exception {
1982    final TableName tableName = name.getTableName();
1983    try (Table ht = TEST_UTIL.createTable(tableName, FAMILY)) {
1984      byte[][] ROWS = makeN(ROW, 10);
1985      byte[][] QUALIFIERS =
1986        { Bytes.toBytes("col0-<d2v1>-<d3v2>"), Bytes.toBytes("col1-<d2v1>-<d3v2>"),
1987          Bytes.toBytes("col2-<d2v1>-<d3v2>"), Bytes.toBytes("col3-<d2v1>-<d3v2>"),
1988          Bytes.toBytes("col4-<d2v1>-<d3v2>"), Bytes.toBytes("col5-<d2v1>-<d3v2>"),
1989          Bytes.toBytes("col6-<d2v1>-<d3v2>"), Bytes.toBytes("col7-<d2v1>-<d3v2>"),
1990          Bytes.toBytes("col8-<d2v1>-<d3v2>"), Bytes.toBytes("col9-<d2v1>-<d3v2>") };
1991      for (int i = 0; i < 10; i++) {
1992        Put put = new Put(ROWS[i]);
1993        put.addColumn(FAMILY, QUALIFIERS[i], VALUE);
1994        ht.put(put);
1995      }
1996      Scan scan = new Scan();
1997      scan.setReversed(true);
1998      scan.addFamily(FAMILY);
1999      Filter filter = new KeyOnlyFilter(true);
2000      scan.setFilter(filter);
2001      try (ResultScanner ignored = ht.getScanner(scan)) {
2002        int count = 0;
2003        for (Result result : ht.getScanner(scan)) {
2004          assertEquals(1, result.size());
2005          assertEquals(Bytes.SIZEOF_INT, result.rawCells()[0].getValueLength());
2006          assertEquals(VALUE.length, Bytes.toInt(CellUtil.cloneValue(result.rawCells()[0])));
2007          count++;
2008        }
2009        assertEquals(10, count);
2010      }
2011    }
2012  }
2013
2014  /**
2015   * Test simple table and non-existent row cases.
2016   */
2017  @Test
2018  public void testSimpleMissingWithReverseScan() throws Exception {
2019    final TableName tableName = name.getTableName();
2020    try (Table ht = TEST_UTIL.createTable(tableName, FAMILY)) {
2021      byte[][] ROWS = makeN(ROW, 4);
2022
2023      // Try to get a row on an empty table
2024      Scan scan = new Scan();
2025      scan.setReversed(true);
2026      Result result = getSingleScanResult(ht, scan);
2027      assertNullResult(result);
2028
2029      scan = new Scan(ROWS[0]);
2030      scan.setReversed(true);
2031      result = getSingleScanResult(ht, scan);
2032      assertNullResult(result);
2033
2034      scan = new Scan(ROWS[0], ROWS[1]);
2035      scan.setReversed(true);
2036      result = getSingleScanResult(ht, scan);
2037      assertNullResult(result);
2038
2039      scan = new Scan();
2040      scan.setReversed(true);
2041      scan.addFamily(FAMILY);
2042      result = getSingleScanResult(ht, scan);
2043      assertNullResult(result);
2044
2045      scan = new Scan();
2046      scan.setReversed(true);
2047      scan.addColumn(FAMILY, QUALIFIER);
2048      result = getSingleScanResult(ht, scan);
2049      assertNullResult(result);
2050
2051      // Insert a row
2052
2053      Put put = new Put(ROWS[2]);
2054      put.addColumn(FAMILY, QUALIFIER, VALUE);
2055      ht.put(put);
2056
2057      // Make sure we can scan the row
2058      scan = new Scan();
2059      scan.setReversed(true);
2060      result = getSingleScanResult(ht, scan);
2061      assertSingleResult(result, ROWS[2], FAMILY, QUALIFIER, VALUE);
2062
2063      scan = new Scan(ROWS[3], ROWS[0]);
2064      scan.setReversed(true);
2065      result = getSingleScanResult(ht, scan);
2066      assertSingleResult(result, ROWS[2], FAMILY, QUALIFIER, VALUE);
2067
2068      scan = new Scan(ROWS[2], ROWS[1]);
2069      scan.setReversed(true);
2070      result = getSingleScanResult(ht, scan);
2071      assertSingleResult(result, ROWS[2], FAMILY, QUALIFIER, VALUE);
2072
2073      // Try to scan empty rows around it
2074      // Introduced MemStore#shouldSeekForReverseScan to fix the following
2075      scan = new Scan(ROWS[1]);
2076      scan.setReversed(true);
2077      result = getSingleScanResult(ht, scan);
2078      assertNullResult(result);
2079    }
2080  }
2081
2082  @Test
2083  public void testNullWithReverseScan() throws Exception {
2084    final TableName tableName = name.getTableName();
2085    try (Table ht = TEST_UTIL.createTable(tableName, FAMILY)) {
2086      // Null qualifier (should work)
2087      Put put = new Put(ROW);
2088      put.addColumn(FAMILY, null, VALUE);
2089      ht.put(put);
2090      scanTestNull(ht, ROW, FAMILY, VALUE, true);
2091      Delete delete = new Delete(ROW);
2092      delete.addColumns(FAMILY, null);
2093      ht.delete(delete);
2094    }
2095
2096    // Use a new table
2097    try (Table ht =
2098      TEST_UTIL.createTable(TableName.valueOf(name.getTableName().toString() + "2"), FAMILY)) {
2099      // Empty qualifier, byte[0] instead of null (should work)
2100      Put put = new Put(ROW);
2101      put.addColumn(FAMILY, HConstants.EMPTY_BYTE_ARRAY, VALUE);
2102      ht.put(put);
2103      scanTestNull(ht, ROW, FAMILY, VALUE, true);
2104      TEST_UTIL.flush();
2105      scanTestNull(ht, ROW, FAMILY, VALUE, true);
2106      Delete delete = new Delete(ROW);
2107      delete.addColumns(FAMILY, HConstants.EMPTY_BYTE_ARRAY);
2108      ht.delete(delete);
2109      // Null value
2110      put = new Put(ROW);
2111      put.addColumn(FAMILY, QUALIFIER, null);
2112      ht.put(put);
2113      Scan scan = new Scan();
2114      scan.setReversed(true);
2115      scan.addColumn(FAMILY, QUALIFIER);
2116      Result result = getSingleScanResult(ht, scan);
2117      assertSingleResult(result, ROW, FAMILY, QUALIFIER, null);
2118    }
2119  }
2120
2121  @Test
2122  @SuppressWarnings("checkstyle:MethodLength")
2123  public void testDeletesWithReverseScan() throws Exception {
2124    final TableName tableName = name.getTableName();
2125    byte[][] ROWS = makeNAscii(ROW, 6);
2126    byte[][] FAMILIES = makeNAscii(FAMILY, 3);
2127    byte[][] VALUES = makeN(VALUE, 5);
2128    long[] ts = { 1000, 2000, 3000, 4000, 5000 };
2129    try (Table ht = TEST_UTIL.createTable(tableName, FAMILIES, 3)) {
2130
2131      Put put = new Put(ROW);
2132      put.addColumn(FAMILIES[0], QUALIFIER, ts[0], VALUES[0]);
2133      put.addColumn(FAMILIES[0], QUALIFIER, ts[1], VALUES[1]);
2134      ht.put(put);
2135
2136      Delete delete = new Delete(ROW);
2137      delete.addFamily(FAMILIES[0], ts[0]);
2138      ht.delete(delete);
2139
2140      Scan scan = new Scan(ROW);
2141      scan.setReversed(true);
2142      scan.addFamily(FAMILIES[0]);
2143      scan.setMaxVersions(Integer.MAX_VALUE);
2144      Result result = getSingleScanResult(ht, scan);
2145      assertNResult(result, ROW, FAMILIES[0], QUALIFIER, new long[] { ts[1] },
2146        new byte[][] { VALUES[1] }, 0, 0);
2147
2148      // Test delete latest version
2149      put = new Put(ROW);
2150      put.addColumn(FAMILIES[0], QUALIFIER, ts[4], VALUES[4]);
2151      put.addColumn(FAMILIES[0], QUALIFIER, ts[2], VALUES[2]);
2152      put.addColumn(FAMILIES[0], QUALIFIER, ts[3], VALUES[3]);
2153      put.addColumn(FAMILIES[0], null, ts[4], VALUES[4]);
2154      put.addColumn(FAMILIES[0], null, ts[2], VALUES[2]);
2155      put.addColumn(FAMILIES[0], null, ts[3], VALUES[3]);
2156      ht.put(put);
2157
2158      delete = new Delete(ROW);
2159      delete.addColumn(FAMILIES[0], QUALIFIER); // ts[4]
2160      ht.delete(delete);
2161
2162      scan = new Scan(ROW);
2163      scan.setReversed(true);
2164      scan.addColumn(FAMILIES[0], QUALIFIER);
2165      scan.setMaxVersions(Integer.MAX_VALUE);
2166      result = getSingleScanResult(ht, scan);
2167      assertNResult(result, ROW, FAMILIES[0], QUALIFIER, new long[] { ts[1], ts[2], ts[3] },
2168        new byte[][] { VALUES[1], VALUES[2], VALUES[3] }, 0, 2);
2169
2170      // Test for HBASE-1847
2171      delete = new Delete(ROW);
2172      delete.addColumn(FAMILIES[0], null);
2173      ht.delete(delete);
2174
2175      // Cleanup null qualifier
2176      delete = new Delete(ROW);
2177      delete.addColumns(FAMILIES[0], null);
2178      ht.delete(delete);
2179
2180      // Expected client behavior might be that you can re-put deleted values
2181      // But alas, this is not to be. We can't put them back in either case.
2182
2183      put = new Put(ROW);
2184      put.addColumn(FAMILIES[0], QUALIFIER, ts[0], VALUES[0]);
2185      put.addColumn(FAMILIES[0], QUALIFIER, ts[4], VALUES[4]);
2186      ht.put(put);
2187
2188      // The Scanner returns the previous values, the expected-naive-unexpected
2189      // behavior
2190
2191      scan = new Scan(ROW);
2192      scan.setReversed(true);
2193      scan.addFamily(FAMILIES[0]);
2194      scan.setMaxVersions(Integer.MAX_VALUE);
2195      result = getSingleScanResult(ht, scan);
2196      assertNResult(result, ROW, FAMILIES[0], QUALIFIER, new long[] { ts[1], ts[2], ts[3] },
2197        new byte[][] { VALUES[1], VALUES[2], VALUES[3] }, 0, 2);
2198
2199      // Test deleting an entire family from one row but not the other various
2200      // ways
2201
2202      put = new Put(ROWS[0]);
2203      put.addColumn(FAMILIES[1], QUALIFIER, ts[0], VALUES[0]);
2204      put.addColumn(FAMILIES[1], QUALIFIER, ts[1], VALUES[1]);
2205      put.addColumn(FAMILIES[2], QUALIFIER, ts[2], VALUES[2]);
2206      put.addColumn(FAMILIES[2], QUALIFIER, ts[3], VALUES[3]);
2207      ht.put(put);
2208
2209      put = new Put(ROWS[1]);
2210      put.addColumn(FAMILIES[1], QUALIFIER, ts[0], VALUES[0]);
2211      put.addColumn(FAMILIES[1], QUALIFIER, ts[1], VALUES[1]);
2212      put.addColumn(FAMILIES[2], QUALIFIER, ts[2], VALUES[2]);
2213      put.addColumn(FAMILIES[2], QUALIFIER, ts[3], VALUES[3]);
2214      ht.put(put);
2215
2216      put = new Put(ROWS[2]);
2217      put.addColumn(FAMILIES[1], QUALIFIER, ts[0], VALUES[0]);
2218      put.addColumn(FAMILIES[1], QUALIFIER, ts[1], VALUES[1]);
2219      put.addColumn(FAMILIES[2], QUALIFIER, ts[2], VALUES[2]);
2220      put.addColumn(FAMILIES[2], QUALIFIER, ts[3], VALUES[3]);
2221      ht.put(put);
2222
2223      delete = new Delete(ROWS[0]);
2224      delete.addFamily(FAMILIES[2]);
2225      ht.delete(delete);
2226
2227      delete = new Delete(ROWS[1]);
2228      delete.addColumns(FAMILIES[1], QUALIFIER);
2229      ht.delete(delete);
2230
2231      delete = new Delete(ROWS[2]);
2232      delete.addColumn(FAMILIES[1], QUALIFIER);
2233      delete.addColumn(FAMILIES[1], QUALIFIER);
2234      delete.addColumn(FAMILIES[2], QUALIFIER);
2235      ht.delete(delete);
2236
2237      scan = new Scan(ROWS[0]);
2238      scan.setReversed(true);
2239      scan.addFamily(FAMILIES[1]);
2240      scan.addFamily(FAMILIES[2]);
2241      scan.setMaxVersions(Integer.MAX_VALUE);
2242      result = getSingleScanResult(ht, scan);
2243      assertEquals("Expected 2 keys but received " + result.size(), 2, result.size());
2244      assertNResult(result, ROWS[0], FAMILIES[1], QUALIFIER, new long[] { ts[0], ts[1] },
2245        new byte[][] { VALUES[0], VALUES[1] }, 0, 1);
2246
2247      scan = new Scan(ROWS[1]);
2248      scan.setReversed(true);
2249      scan.addFamily(FAMILIES[1]);
2250      scan.addFamily(FAMILIES[2]);
2251      scan.setMaxVersions(Integer.MAX_VALUE);
2252      result = getSingleScanResult(ht, scan);
2253      assertEquals("Expected 2 keys but received " + result.size(), 2, result.size());
2254
2255      scan = new Scan(ROWS[2]);
2256      scan.setReversed(true);
2257      scan.addFamily(FAMILIES[1]);
2258      scan.addFamily(FAMILIES[2]);
2259      scan.setMaxVersions(Integer.MAX_VALUE);
2260      result = getSingleScanResult(ht, scan);
2261      assertEquals(1, result.size());
2262      assertNResult(result, ROWS[2], FAMILIES[2], QUALIFIER, new long[] { ts[2] },
2263        new byte[][] { VALUES[2] }, 0, 0);
2264
2265      // Test if we delete the family first in one row (HBASE-1541)
2266
2267      delete = new Delete(ROWS[3]);
2268      delete.addFamily(FAMILIES[1]);
2269      ht.delete(delete);
2270
2271      put = new Put(ROWS[3]);
2272      put.addColumn(FAMILIES[2], QUALIFIER, VALUES[0]);
2273      ht.put(put);
2274
2275      put = new Put(ROWS[4]);
2276      put.addColumn(FAMILIES[1], QUALIFIER, VALUES[1]);
2277      put.addColumn(FAMILIES[2], QUALIFIER, VALUES[2]);
2278      ht.put(put);
2279
2280      scan = new Scan(ROWS[4]);
2281      scan.setReversed(true);
2282      scan.addFamily(FAMILIES[1]);
2283      scan.addFamily(FAMILIES[2]);
2284      scan.setMaxVersions(Integer.MAX_VALUE);
2285      ResultScanner scanner = ht.getScanner(scan);
2286      result = scanner.next();
2287      assertEquals("Expected 2 keys but received " + result.size(), 2, result.size());
2288      assertTrue(Bytes.equals(CellUtil.cloneRow(result.rawCells()[0]), ROWS[4]));
2289      assertTrue(Bytes.equals(CellUtil.cloneRow(result.rawCells()[1]), ROWS[4]));
2290      assertTrue(Bytes.equals(CellUtil.cloneValue(result.rawCells()[0]), VALUES[1]));
2291      assertTrue(Bytes.equals(CellUtil.cloneValue(result.rawCells()[1]), VALUES[2]));
2292      result = scanner.next();
2293      assertEquals("Expected 1 key but received " + result.size(), 1, result.size());
2294      assertTrue(Bytes.equals(CellUtil.cloneRow(result.rawCells()[0]), ROWS[3]));
2295      assertTrue(Bytes.equals(CellUtil.cloneValue(result.rawCells()[0]), VALUES[0]));
2296      scanner.close();
2297    }
2298  }
2299
2300  /**
2301   * Tests reversed scan under multi regions
2302   */
2303  @Test
2304  public void testReversedScanUnderMultiRegions() throws Exception {
2305    // Test Initialization.
2306    final TableName tableName = name.getTableName();
2307    byte[] maxByteArray = ConnectionUtils.MAX_BYTE_ARRAY;
2308    byte[][] splitRows = new byte[][] { Bytes.toBytes("005"),
2309      Bytes.add(Bytes.toBytes("005"), Bytes.multiple(maxByteArray, 16)), Bytes.toBytes("006"),
2310      Bytes.add(Bytes.toBytes("006"), Bytes.multiple(maxByteArray, 8)), Bytes.toBytes("007"),
2311      Bytes.add(Bytes.toBytes("007"), Bytes.multiple(maxByteArray, 4)), Bytes.toBytes("008"),
2312      Bytes.multiple(maxByteArray, 2) };
2313    try (Table table = TEST_UTIL.createTable(tableName, FAMILY, splitRows)) {
2314      TEST_UTIL.waitUntilAllRegionsAssigned(table.getName());
2315
2316      try (RegionLocator l = TEST_UTIL.getConnection().getRegionLocator(tableName)) {
2317        assertEquals(splitRows.length + 1, l.getAllRegionLocations().size());
2318      }
2319      // Insert one row each region
2320      int insertNum = splitRows.length;
2321      for (byte[] splitRow : splitRows) {
2322        Put put = new Put(splitRow);
2323        put.addColumn(FAMILY, QUALIFIER, VALUE);
2324        table.put(put);
2325      }
2326
2327      // scan forward
2328      try (ResultScanner scanner = table.getScanner(new Scan())) {
2329        int count = 0;
2330        for (Result r : scanner) {
2331          assertFalse(r.isEmpty());
2332          count++;
2333        }
2334        assertEquals(insertNum, count);
2335      }
2336
2337      // scan backward
2338      Scan scan = new Scan();
2339      scan.setReversed(true);
2340      try (ResultScanner scanner = table.getScanner(scan)) {
2341        int count = 0;
2342        byte[] lastRow = null;
2343        for (Result r : scanner) {
2344          assertFalse(r.isEmpty());
2345          count++;
2346          byte[] thisRow = r.getRow();
2347          if (lastRow != null) {
2348            assertTrue("Error scan order, last row= " + Bytes.toString(lastRow) + ",this row="
2349              + Bytes.toString(thisRow), Bytes.compareTo(thisRow, lastRow) < 0);
2350          }
2351          lastRow = thisRow;
2352        }
2353        assertEquals(insertNum, count);
2354      }
2355    }
2356  }
2357
2358  /**
2359   * Tests reversed scan under multi regions
2360   */
2361  @Test
2362  public void testSmallReversedScanUnderMultiRegions() throws Exception {
2363    // Test Initialization.
2364    final TableName tableName = name.getTableName();
2365    byte[][] splitRows = new byte[][] { Bytes.toBytes("000"), Bytes.toBytes("002"),
2366      Bytes.toBytes("004"), Bytes.toBytes("006"), Bytes.toBytes("008"), Bytes.toBytes("010") };
2367    try (Table table = TEST_UTIL.createTable(tableName, FAMILY, splitRows)) {
2368      TEST_UTIL.waitUntilAllRegionsAssigned(table.getName());
2369
2370      try (RegionLocator l = TEST_UTIL.getConnection().getRegionLocator(tableName)) {
2371        assertEquals(splitRows.length + 1, l.getAllRegionLocations().size());
2372      }
2373      for (byte[] splitRow : splitRows) {
2374        Put put = new Put(splitRow);
2375        put.addColumn(FAMILY, QUALIFIER, VALUE);
2376        table.put(put);
2377
2378        byte[] nextRow = Bytes.copy(splitRow);
2379        nextRow[nextRow.length - 1]++;
2380
2381        put = new Put(nextRow);
2382        put.addColumn(FAMILY, QUALIFIER, VALUE);
2383        table.put(put);
2384      }
2385
2386      // scan forward
2387      try (ResultScanner scanner = table.getScanner(new Scan())) {
2388        int count = 0;
2389        for (Result r : scanner) {
2390          assertTrue(!r.isEmpty());
2391          count++;
2392        }
2393        assertEquals(12, count);
2394      }
2395
2396      reverseScanTest(table, false);
2397      reverseScanTest(table, true);
2398    }
2399  }
2400
2401  private void reverseScanTest(Table table, boolean small) throws IOException {
2402    // scan backward
2403    Scan scan = new Scan();
2404    scan.setReversed(true);
2405    try (ResultScanner scanner = table.getScanner(scan)) {
2406      int count = 0;
2407      byte[] lastRow = null;
2408      for (Result r : scanner) {
2409        assertTrue(!r.isEmpty());
2410        count++;
2411        byte[] thisRow = r.getRow();
2412        if (lastRow != null) {
2413          assertTrue("Error scan order, last row= " + Bytes.toString(lastRow) + ",this row="
2414            + Bytes.toString(thisRow), Bytes.compareTo(thisRow, lastRow) < 0);
2415        }
2416        lastRow = thisRow;
2417      }
2418      assertEquals(12, count);
2419    }
2420
2421    scan = new Scan();
2422    scan.setSmall(small);
2423    scan.setReversed(true);
2424    scan.setStartRow(Bytes.toBytes("002"));
2425    try (ResultScanner scanner = table.getScanner(scan)) {
2426      int count = 0;
2427      byte[] lastRow = null;
2428      for (Result r : scanner) {
2429        assertTrue(!r.isEmpty());
2430        count++;
2431        byte[] thisRow = r.getRow();
2432        if (lastRow != null) {
2433          assertTrue("Error scan order, last row= " + Bytes.toString(lastRow) + ",this row="
2434            + Bytes.toString(thisRow), Bytes.compareTo(thisRow, lastRow) < 0);
2435        }
2436        lastRow = thisRow;
2437      }
2438      assertEquals(3, count); // 000 001 002
2439    }
2440
2441    scan = new Scan();
2442    scan.setSmall(small);
2443    scan.setReversed(true);
2444    scan.setStartRow(Bytes.toBytes("002"));
2445    scan.setStopRow(Bytes.toBytes("000"));
2446    try (ResultScanner scanner = table.getScanner(scan)) {
2447      int count = 0;
2448      byte[] lastRow = null;
2449      for (Result r : scanner) {
2450        assertFalse(r.isEmpty());
2451        count++;
2452        byte[] thisRow = r.getRow();
2453        if (lastRow != null) {
2454          assertTrue("Error scan order, last row= " + Bytes.toString(lastRow) + ",this row="
2455            + Bytes.toString(thisRow), Bytes.compareTo(thisRow, lastRow) < 0);
2456        }
2457        lastRow = thisRow;
2458      }
2459      assertEquals(2, count); // 001 002
2460    }
2461
2462    scan = new Scan();
2463    scan.setSmall(small);
2464    scan.setReversed(true);
2465    scan.setStartRow(Bytes.toBytes("001"));
2466    try (ResultScanner scanner = table.getScanner(scan)) {
2467      int count = 0;
2468      byte[] lastRow = null;
2469      for (Result r : scanner) {
2470        assertFalse(r.isEmpty());
2471        count++;
2472        byte[] thisRow = r.getRow();
2473        if (lastRow != null) {
2474          assertTrue("Error scan order, last row= " + Bytes.toString(lastRow) + ",this row="
2475            + Bytes.toString(thisRow), Bytes.compareTo(thisRow, lastRow) < 0);
2476        }
2477        lastRow = thisRow;
2478      }
2479      assertEquals(2, count); // 000 001
2480    }
2481
2482    scan = new Scan();
2483    scan.setSmall(small);
2484    scan.setReversed(true);
2485    scan.setStartRow(Bytes.toBytes("000"));
2486    try (ResultScanner scanner = table.getScanner(scan)) {
2487      int count = 0;
2488      byte[] lastRow = null;
2489      for (Result r : scanner) {
2490        assertFalse(r.isEmpty());
2491        count++;
2492        byte[] thisRow = r.getRow();
2493        if (lastRow != null) {
2494          assertTrue("Error scan order, last row= " + Bytes.toString(lastRow) + ",this row="
2495            + Bytes.toString(thisRow), Bytes.compareTo(thisRow, lastRow) < 0);
2496        }
2497        lastRow = thisRow;
2498      }
2499      assertEquals(1, count); // 000
2500    }
2501
2502    scan = new Scan();
2503    scan.setSmall(small);
2504    scan.setReversed(true);
2505    scan.setStartRow(Bytes.toBytes("006"));
2506    scan.setStopRow(Bytes.toBytes("002"));
2507    try (ResultScanner scanner = table.getScanner(scan)) {
2508      int count = 0;
2509      byte[] lastRow = null;
2510      for (Result r : scanner) {
2511        assertFalse(r.isEmpty());
2512        count++;
2513        byte[] thisRow = r.getRow();
2514        if (lastRow != null) {
2515          assertTrue("Error scan order, last row= " + Bytes.toString(lastRow) + ",this row="
2516            + Bytes.toString(thisRow), Bytes.compareTo(thisRow, lastRow) < 0);
2517        }
2518        lastRow = thisRow;
2519      }
2520      assertEquals(4, count); // 003 004 005 006
2521    }
2522  }
2523
2524  @Test
2525  public void testFilterAllRecords() throws IOException {
2526    Scan scan = new Scan();
2527    scan.setBatch(1);
2528    scan.setCaching(1);
2529    // Filter out any records
2530    scan.setFilter(new FilterList(new FirstKeyOnlyFilter(), new InclusiveStopFilter(new byte[0])));
2531    try (Table table = TEST_UTIL.getConnection().getTable(TableName.META_TABLE_NAME)) {
2532      try (ResultScanner s = table.getScanner(scan)) {
2533        assertNull(s.next());
2534      }
2535    }
2536  }
2537
2538  @Test
2539  public void testCellSizeLimit() throws IOException {
2540    final TableName tableName = name.getTableName();
2541    TableDescriptorBuilder.ModifyableTableDescriptor tableDescriptor =
2542      new TableDescriptorBuilder.ModifyableTableDescriptor(tableName)
2543        .setValue(HRegion.HBASE_MAX_CELL_SIZE_KEY, Integer.toString(10 * 1024));
2544    ColumnFamilyDescriptor familyDescriptor =
2545      new ColumnFamilyDescriptorBuilder.ModifyableColumnFamilyDescriptor(FAMILY);
2546
2547    tableDescriptor.setColumnFamily(familyDescriptor);
2548    try (Admin admin = TEST_UTIL.getAdmin()) {
2549      admin.createTable(tableDescriptor);
2550    }
2551    // Will succeed
2552    try (Table t = TEST_UTIL.getConnection().getTable(tableName)) {
2553      t.put(new Put(ROW).addColumn(FAMILY, QUALIFIER, Bytes.toBytes(0L)));
2554      t.increment(new Increment(ROW).addColumn(FAMILY, QUALIFIER, 1L));
2555    }
2556    // Will succeed
2557    try (Table t = TEST_UTIL.getConnection().getTable(tableName)) {
2558      t.put(new Put(ROW).addColumn(FAMILY, QUALIFIER, new byte[9 * 1024]));
2559    }
2560    // Will fail
2561    try (Table t = TEST_UTIL.getConnection().getTable(tableName)) {
2562      try {
2563        t.put(new Put(ROW).addColumn(FAMILY, QUALIFIER, new byte[10 * 1024]));
2564        fail("Oversize cell failed to trigger exception");
2565      } catch (IOException e) {
2566        // expected
2567      }
2568      try {
2569        t.append(new Append(ROW).addColumn(FAMILY, QUALIFIER, new byte[2 * 1024]));
2570        fail("Oversize cell failed to trigger exception");
2571      } catch (IOException e) {
2572        // expected
2573      }
2574    }
2575  }
2576
2577  @Test
2578  public void testCellSizeNoLimit() throws IOException {
2579    final TableName tableName = name.getTableName();
2580    ColumnFamilyDescriptor familyDescriptor =
2581      new ColumnFamilyDescriptorBuilder.ModifyableColumnFamilyDescriptor(FAMILY);
2582    TableDescriptorBuilder.ModifyableTableDescriptor tableDescriptor =
2583      new TableDescriptorBuilder.ModifyableTableDescriptor(tableName)
2584        .setValue(HRegion.HBASE_MAX_CELL_SIZE_KEY, Integer.toString(0));
2585    tableDescriptor.setColumnFamily(familyDescriptor);
2586
2587    try (Admin admin = TEST_UTIL.getAdmin()) {
2588      admin.createTable(tableDescriptor);
2589    }
2590
2591    // Will succeed
2592    try (Table ht = TEST_UTIL.getConnection().getTable(tableName)) {
2593      ht.put(
2594        new Put(ROW).addColumn(FAMILY, QUALIFIER, new byte[HRegion.DEFAULT_MAX_CELL_SIZE - 1024]));
2595      ht.append(new Append(ROW).addColumn(FAMILY, QUALIFIER, new byte[1024 + 1]));
2596    }
2597  }
2598
2599  @Test
2600  public void testDeleteSpecifiedVersionOfSpecifiedColumn() throws Exception {
2601    final TableName tableName = name.getTableName();
2602
2603    byte[][] VALUES = makeN(VALUE, 5);
2604    long[] ts = { 1000, 2000, 3000, 4000, 5000 };
2605
2606    try (Table ht = TEST_UTIL.createTable(tableName, FAMILY, 5)) {
2607
2608      Put put = new Put(ROW);
2609      // Put version 1000,2000,3000,4000 of column FAMILY:QUALIFIER
2610      for (int t = 0; t < 4; t++) {
2611        put.addColumn(FAMILY, QUALIFIER, ts[t], VALUES[t]);
2612      }
2613      ht.put(put);
2614
2615      Delete delete = new Delete(ROW);
2616      // Delete version 3000 of column FAMILY:QUALIFIER
2617      delete.addColumn(FAMILY, QUALIFIER, ts[2]);
2618      ht.delete(delete);
2619
2620      Get get = new Get(ROW);
2621      get.addColumn(FAMILY, QUALIFIER);
2622      get.readVersions(Integer.MAX_VALUE);
2623      Result result = ht.get(get);
2624      // verify version 1000,2000,4000 remains for column FAMILY:QUALIFIER
2625      assertNResult(result, ROW, FAMILY, QUALIFIER, new long[] { ts[0], ts[1], ts[3] },
2626        new byte[][] { VALUES[0], VALUES[1], VALUES[3] }, 0, 2);
2627
2628      delete = new Delete(ROW);
2629      // Delete a version 5000 of column FAMILY:QUALIFIER which didn't exist
2630      delete.addColumn(FAMILY, QUALIFIER, ts[4]);
2631      ht.delete(delete);
2632
2633      get = new Get(ROW);
2634      get.addColumn(FAMILY, QUALIFIER);
2635      get.readVersions(Integer.MAX_VALUE);
2636      result = ht.get(get);
2637      // verify version 1000,2000,4000 remains for column FAMILY:QUALIFIER
2638      assertNResult(result, ROW, FAMILY, QUALIFIER, new long[] { ts[0], ts[1], ts[3] },
2639        new byte[][] { VALUES[0], VALUES[1], VALUES[3] }, 0, 2);
2640    }
2641  }
2642
2643  @Test
2644  public void testDeleteLatestVersionOfSpecifiedColumn() throws Exception {
2645    final TableName tableName = name.getTableName();
2646    byte[][] VALUES = makeN(VALUE, 5);
2647    long[] ts = { 1000, 2000, 3000, 4000, 5000 };
2648    try (Table ht = TEST_UTIL.createTable(tableName, FAMILY, 5)) {
2649      Put put = new Put(ROW);
2650      // Put version 1000,2000,3000,4000 of column FAMILY:QUALIFIER
2651      for (int t = 0; t < 4; t++) {
2652        put.addColumn(FAMILY, QUALIFIER, ts[t], VALUES[t]);
2653      }
2654      ht.put(put);
2655
2656      Delete delete = new Delete(ROW);
2657      // Delete latest version of column FAMILY:QUALIFIER
2658      delete.addColumn(FAMILY, QUALIFIER);
2659      ht.delete(delete);
2660
2661      Get get = new Get(ROW);
2662      get.addColumn(FAMILY, QUALIFIER);
2663      get.readVersions(Integer.MAX_VALUE);
2664      Result result = ht.get(get);
2665      // verify version 1000,2000,3000 remains for column FAMILY:QUALIFIER
2666      assertNResult(result, ROW, FAMILY, QUALIFIER, new long[] { ts[0], ts[1], ts[2] },
2667        new byte[][] { VALUES[0], VALUES[1], VALUES[2] }, 0, 2);
2668
2669      delete = new Delete(ROW);
2670      // Delete two latest version of column FAMILY:QUALIFIER
2671      delete.addColumn(FAMILY, QUALIFIER);
2672      delete.addColumn(FAMILY, QUALIFIER);
2673      ht.delete(delete);
2674
2675      get = new Get(ROW);
2676      get.addColumn(FAMILY, QUALIFIER);
2677      get.readVersions(Integer.MAX_VALUE);
2678      result = ht.get(get);
2679      // verify version 1000 remains for column FAMILY:QUALIFIER
2680      assertNResult(result, ROW, FAMILY, QUALIFIER, new long[] { ts[0] },
2681        new byte[][] { VALUES[0] }, 0, 0);
2682
2683      put = new Put(ROW);
2684      // Put a version 5000 of column FAMILY:QUALIFIER
2685      put.addColumn(FAMILY, QUALIFIER, ts[4], VALUES[4]);
2686      ht.put(put);
2687
2688      get = new Get(ROW);
2689      get.addColumn(FAMILY, QUALIFIER);
2690      get.readVersions(Integer.MAX_VALUE);
2691      result = ht.get(get);
2692      // verify version 1000,5000 remains for column FAMILY:QUALIFIER
2693      assertNResult(result, ROW, FAMILY, QUALIFIER, new long[] { ts[0], ts[4] },
2694        new byte[][] { VALUES[0], VALUES[4] }, 0, 1);
2695    }
2696  }
2697
2698  /**
2699   * Test for HBASE-17125
2700   */
2701  @Test
2702  public void testReadWithFilter() throws Exception {
2703    final TableName tableName = name.getTableName();
2704    try (Table table = TEST_UTIL.createTable(tableName, FAMILY, 3)) {
2705
2706      byte[] VALUEA = Bytes.toBytes("value-a");
2707      byte[] VALUEB = Bytes.toBytes("value-b");
2708      long[] ts = { 1000, 2000, 3000, 4000 };
2709
2710      Put put = new Put(ROW);
2711      // Put version 1000,2000,3000,4000 of column FAMILY:QUALIFIER
2712      for (int t = 0; t <= 3; t++) {
2713        if (t <= 1) {
2714          put.addColumn(FAMILY, QUALIFIER, ts[t], VALUEA);
2715        } else {
2716          put.addColumn(FAMILY, QUALIFIER, ts[t], VALUEB);
2717        }
2718      }
2719      table.put(put);
2720
2721      Scan scan = new Scan()
2722        .setFilter(new ValueFilter(CompareOperator.EQUAL, new SubstringComparator("value-a")))
2723        .setMaxVersions(3);
2724      ResultScanner scanner = table.getScanner(scan);
2725      Result result = scanner.next();
2726      // ts[0] has gone from user view. Only read ts[2] which value is less or equal to 3
2727      assertNResult(result, ROW, FAMILY, QUALIFIER, new long[] { ts[1] }, new byte[][] { VALUEA },
2728        0, 0);
2729
2730      Get get = new Get(ROW)
2731        .setFilter(new ValueFilter(CompareOperator.EQUAL, new SubstringComparator("value-a")))
2732        .readVersions(3);
2733      result = table.get(get);
2734      // ts[0] has gone from user view. Only read ts[2] which value is less or equal to 3
2735      assertNResult(result, ROW, FAMILY, QUALIFIER, new long[] { ts[1] }, new byte[][] { VALUEA },
2736        0, 0);
2737
2738      // Test with max versions 1, it should still read ts[1]
2739      scan = new Scan()
2740        .setFilter(new ValueFilter(CompareOperator.EQUAL, new SubstringComparator("value-a")))
2741        .setMaxVersions(1);
2742      scanner = table.getScanner(scan);
2743      result = scanner.next();
2744      // ts[0] has gone from user view. Only read ts[2] which value is less or equal to 3
2745      assertNResult(result, ROW, FAMILY, QUALIFIER, new long[] { ts[1] }, new byte[][] { VALUEA },
2746        0, 0);
2747
2748      // Test with max versions 1, it should still read ts[1]
2749      get = new Get(ROW)
2750        .setFilter(new ValueFilter(CompareOperator.EQUAL, new SubstringComparator("value-a")))
2751        .readVersions(1);
2752      result = table.get(get);
2753      // ts[0] has gone from user view. Only read ts[2] which value is less or equal to 3
2754      assertNResult(result, ROW, FAMILY, QUALIFIER, new long[] { ts[1] }, new byte[][] { VALUEA },
2755        0, 0);
2756
2757      // Test with max versions 5, it should still read ts[1]
2758      scan = new Scan()
2759        .setFilter(new ValueFilter(CompareOperator.EQUAL, new SubstringComparator("value-a")))
2760        .setMaxVersions(5);
2761      scanner = table.getScanner(scan);
2762      result = scanner.next();
2763      // ts[0] has gone from user view. Only read ts[2] which value is less or equal to 3
2764      assertNResult(result, ROW, FAMILY, QUALIFIER, new long[] { ts[1] }, new byte[][] { VALUEA },
2765        0, 0);
2766
2767      // Test with max versions 5, it should still read ts[1]
2768      get = new Get(ROW)
2769        .setFilter(new ValueFilter(CompareOperator.EQUAL, new SubstringComparator("value-a")))
2770        .readVersions(5);
2771      result = table.get(get);
2772      // ts[0] has gone from user view. Only read ts[2] which value is less or equal to 3
2773      assertNResult(result, ROW, FAMILY, QUALIFIER, new long[] { ts[1] }, new byte[][] { VALUEA },
2774        0, 0);
2775    }
2776  }
2777
2778  @Test
2779  public void testCellUtilTypeMethods() throws IOException {
2780    final TableName tableName = name.getTableName();
2781    try (Table table = TEST_UTIL.createTable(tableName, FAMILY)) {
2782
2783      final byte[] row = Bytes.toBytes("p");
2784      Put p = new Put(row);
2785      p.addColumn(FAMILY, QUALIFIER, VALUE);
2786      table.put(p);
2787
2788      try (ResultScanner scanner = table.getScanner(new Scan())) {
2789        Result result = scanner.next();
2790        assertNotNull(result);
2791        CellScanner cs = result.cellScanner();
2792        assertTrue(cs.advance());
2793        Cell c = cs.current();
2794        assertTrue(CellUtil.isPut(c));
2795        assertFalse(CellUtil.isDelete(c));
2796        assertFalse(cs.advance());
2797        assertNull(scanner.next());
2798      }
2799
2800      Delete d = new Delete(row);
2801      d.addColumn(FAMILY, QUALIFIER);
2802      table.delete(d);
2803
2804      Scan scan = new Scan();
2805      scan.setRaw(true);
2806      try (ResultScanner scanner = table.getScanner(scan)) {
2807        Result result = scanner.next();
2808        assertNotNull(result);
2809        CellScanner cs = result.cellScanner();
2810        assertTrue(cs.advance());
2811
2812        // First cell should be the delete (masking the Put)
2813        Cell c = cs.current();
2814        assertTrue("Cell should be a Delete: " + c, CellUtil.isDelete(c));
2815        assertFalse("Cell should not be a Put: " + c, CellUtil.isPut(c));
2816
2817        // Second cell should be the original Put
2818        assertTrue(cs.advance());
2819        c = cs.current();
2820        assertFalse("Cell should not be a Delete: " + c, CellUtil.isDelete(c));
2821        assertTrue("Cell should be a Put: " + c, CellUtil.isPut(c));
2822
2823        // No more cells in this row
2824        assertFalse(cs.advance());
2825
2826        // No more results in this scan
2827        assertNull(scanner.next());
2828      }
2829    }
2830  }
2831
2832  @Test(expected = DoNotRetryIOException.class)
2833  public void testCreateTableWithZeroRegionReplicas() throws Exception {
2834    TableName tableName = name.getTableName();
2835    TableDescriptor desc = TableDescriptorBuilder.newBuilder(tableName)
2836      .setColumnFamily(ColumnFamilyDescriptorBuilder.of(Bytes.toBytes("cf")))
2837      .setRegionReplication(0).build();
2838
2839    TEST_UTIL.getAdmin().createTable(desc);
2840  }
2841
2842  @Test(expected = DoNotRetryIOException.class)
2843  public void testModifyTableWithZeroRegionReplicas() throws Exception {
2844    TableName tableName = name.getTableName();
2845    TableDescriptor desc = TableDescriptorBuilder.newBuilder(tableName)
2846      .setColumnFamily(ColumnFamilyDescriptorBuilder.of(Bytes.toBytes("cf"))).build();
2847
2848    TEST_UTIL.getAdmin().createTable(desc);
2849    TableDescriptor newDesc =
2850      TableDescriptorBuilder.newBuilder(desc).setRegionReplication(0).build();
2851
2852    TEST_UTIL.getAdmin().modifyTable(newDesc);
2853  }
2854
2855  @Test(timeout = 60000)
2856  public void testModifyTableWithMemstoreData() throws Exception {
2857    TableName tableName = name.getTableName();
2858    createTableAndValidateTableSchemaModification(tableName, true);
2859  }
2860
2861  @Test(timeout = 60000)
2862  public void testDeleteCFWithMemstoreData() throws Exception {
2863    TableName tableName = name.getTableName();
2864    createTableAndValidateTableSchemaModification(tableName, false);
2865  }
2866
2867  /**
2868   * Create table and validate online schema modification
2869   * @param tableName   Table name
2870   * @param modifyTable Modify table if true otherwise delete column family
2871   * @throws IOException in case of failures
2872   */
2873  private void createTableAndValidateTableSchemaModification(TableName tableName,
2874    boolean modifyTable) throws Exception {
2875    Admin admin = TEST_UTIL.getAdmin();
2876    // Create table with two Cfs
2877    byte[] cf1 = Bytes.toBytes("cf1");
2878    byte[] cf2 = Bytes.toBytes("cf2");
2879    TableDescriptor tableDesc = TableDescriptorBuilder.newBuilder(tableName)
2880      .setColumnFamily(ColumnFamilyDescriptorBuilder.of(cf1))
2881      .setColumnFamily(ColumnFamilyDescriptorBuilder.of(cf2)).build();
2882    admin.createTable(tableDesc);
2883
2884    Table t = TEST_UTIL.getConnection().getTable(tableName);
2885    // Insert few records and flush the table
2886    t.put(new Put(ROW).addColumn(cf1, QUALIFIER, Bytes.toBytes("val1")));
2887    t.put(new Put(ROW).addColumn(cf2, QUALIFIER, Bytes.toBytes("val2")));
2888    admin.flush(tableName);
2889    Path tableDir = CommonFSUtils.getTableDir(TEST_UTIL.getDefaultRootDirPath(), tableName);
2890    List<Path> regionDirs = FSUtils.getRegionDirs(TEST_UTIL.getTestFileSystem(), tableDir);
2891    assertEquals(1, regionDirs.size());
2892    List<Path> familyDirs = FSUtils.getFamilyDirs(TEST_UTIL.getTestFileSystem(), regionDirs.get(0));
2893    assertEquals(2, familyDirs.size());
2894
2895    // Insert record but dont flush the table
2896    t.put(new Put(ROW).addColumn(cf1, QUALIFIER, Bytes.toBytes("val2")));
2897    t.put(new Put(ROW).addColumn(cf2, QUALIFIER, Bytes.toBytes("val2")));
2898
2899    if (modifyTable) {
2900      tableDesc = TableDescriptorBuilder.newBuilder(tableDesc).removeColumnFamily(cf2).build();
2901      admin.modifyTable(tableDesc);
2902    } else {
2903      admin.deleteColumnFamily(tableName, cf2);
2904    }
2905    // After table modification or delete family there should be only one CF in FS
2906    familyDirs = FSUtils.getFamilyDirs(TEST_UTIL.getTestFileSystem(), regionDirs.get(0));
2907    assertEquals("CF dir count should be 1, but was " + familyDirs.size(), 1, familyDirs.size());
2908  }
2909}