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.quotas;
019
020import static org.junit.Assert.assertEquals;
021import static org.junit.Assert.assertNotNull;
022import static org.junit.Assert.assertTrue;
023
024import java.io.IOException;
025import java.util.Map;
026import java.util.concurrent.atomic.AtomicInteger;
027import java.util.concurrent.atomic.AtomicLong;
028import org.apache.hadoop.conf.Configuration;
029import org.apache.hadoop.hbase.Cell;
030import org.apache.hadoop.hbase.CellScanner;
031import org.apache.hadoop.hbase.HBaseClassTestRule;
032import org.apache.hadoop.hbase.HBaseTestingUtility;
033import org.apache.hadoop.hbase.TableName;
034import org.apache.hadoop.hbase.Waiter.Predicate;
035import org.apache.hadoop.hbase.client.Admin;
036import org.apache.hadoop.hbase.client.Connection;
037import org.apache.hadoop.hbase.client.Result;
038import org.apache.hadoop.hbase.client.ResultScanner;
039import org.apache.hadoop.hbase.client.Scan;
040import org.apache.hadoop.hbase.client.SnapshotType;
041import org.apache.hadoop.hbase.client.Table;
042import org.apache.hadoop.hbase.quotas.SpaceQuotaHelperForTests.SpaceQuotaSnapshotPredicate;
043import org.apache.hadoop.hbase.testclassification.LargeTests;
044import org.junit.AfterClass;
045import org.junit.Before;
046import org.junit.BeforeClass;
047import org.junit.ClassRule;
048import org.junit.Rule;
049import org.junit.Test;
050import org.junit.experimental.categories.Category;
051import org.junit.rules.TestName;
052import org.slf4j.Logger;
053import org.slf4j.LoggerFactory;
054
055import org.apache.hbase.thirdparty.com.google.common.collect.Iterables;
056import org.apache.hbase.thirdparty.com.google.protobuf.UnsafeByteOperations;
057
058import org.apache.hadoop.hbase.shaded.protobuf.generated.QuotaProtos;
059
060/**
061 * Test class to exercise the inclusion of snapshots in space quotas
062 */
063@Category({LargeTests.class})
064public class TestSpaceQuotasWithSnapshots {
065
066  @ClassRule
067  public static final HBaseClassTestRule CLASS_RULE =
068      HBaseClassTestRule.forClass(TestSpaceQuotasWithSnapshots.class);
069
070  private static final Logger LOG = LoggerFactory.getLogger(TestSpaceQuotasWithSnapshots.class);
071  private static final HBaseTestingUtility TEST_UTIL = new HBaseTestingUtility();
072  // Global for all tests in the class
073  private static final AtomicLong COUNTER = new AtomicLong(0);
074  private static final long FUDGE_FOR_TABLE_SIZE = 500L * SpaceQuotaHelperForTests.ONE_KILOBYTE;
075
076  @Rule
077  public TestName testName = new TestName();
078  private SpaceQuotaHelperForTests helper;
079  private Connection conn;
080  private Admin admin;
081
082  @BeforeClass
083  public static void setUp() throws Exception {
084    Configuration conf = TEST_UTIL.getConfiguration();
085    SpaceQuotaHelperForTests.updateConfigForQuotas(conf);
086    TEST_UTIL.startMiniCluster(1);
087  }
088
089  @AfterClass
090  public static void tearDown() throws Exception {
091    TEST_UTIL.shutdownMiniCluster();
092  }
093
094  @Before
095  public void removeAllQuotas() throws Exception {
096    helper = new SpaceQuotaHelperForTests(TEST_UTIL, testName, COUNTER);
097    conn = TEST_UTIL.getConnection();
098    admin = TEST_UTIL.getAdmin();
099  }
100
101  @Test
102  public void testTablesInheritSnapshotSize() throws Exception {
103    TableName tn = helper.createTableWithRegions(1);
104    LOG.info("Writing data");
105    // Set a quota
106    QuotaSettings settings = QuotaSettingsFactory.limitTableSpace(
107        tn, SpaceQuotaHelperForTests.ONE_GIGABYTE, SpaceViolationPolicy.NO_INSERTS);
108    admin.setQuota(settings);
109    // Write some data
110    final long initialSize = 2L * SpaceQuotaHelperForTests.ONE_MEGABYTE;
111    helper.writeData(tn, initialSize);
112
113    LOG.info("Waiting until table size reflects written data");
114    // Wait until that data is seen by the master
115    TEST_UTIL.waitFor(30 * 1000, 500, new SpaceQuotaSnapshotPredicate(conn, tn) {
116      @Override boolean evaluate(SpaceQuotaSnapshot snapshot) throws Exception {
117        return snapshot.getUsage() >= initialSize;
118      }
119    });
120
121    // Make sure we see the final quota usage size
122    waitForStableQuotaSize(conn, tn, null);
123
124    // The actual size on disk after we wrote our data the first time
125    final long actualInitialSize = QuotaTableUtil.getCurrentSnapshot(conn, tn).getUsage();
126    LOG.info("Initial table size was " + actualInitialSize);
127
128    LOG.info("Snapshot the table");
129    final String snapshot1 = tn.toString() + "_snapshot1";
130    admin.snapshot(snapshot1, tn);
131
132    // Write the same data again, then flush+compact. This should make sure that
133    // the snapshot is referencing files that the table no longer references.
134    LOG.info("Write more data");
135    helper.writeData(tn, initialSize);
136    LOG.info("Flush the table");
137    admin.flush(tn);
138    LOG.info("Synchronously compacting the table");
139    TEST_UTIL.compact(tn, true);
140
141    final long upperBound = initialSize + FUDGE_FOR_TABLE_SIZE;
142    final long lowerBound = initialSize - FUDGE_FOR_TABLE_SIZE;
143
144    // Store the actual size after writing more data and then compacting it down to one file
145    LOG.info("Waiting for the region reports to reflect the correct size, between ("
146        + lowerBound + ", " + upperBound + ")");
147    TEST_UTIL.waitFor(30 * 1000, 500, new Predicate<Exception>() {
148      @Override
149      public boolean evaluate() throws Exception {
150        long size = getRegionSizeReportForTable(conn, tn);
151        return size < upperBound && size > lowerBound;
152      }
153    });
154
155    // Make sure we see the "final" new size for the table, not some intermediate
156    waitForStableRegionSizeReport(conn, tn);
157    final long finalSize = getRegionSizeReportForTable(conn, tn);
158    assertNotNull("Did not expect to see a null size", finalSize);
159    LOG.info("Last seen size: " + finalSize);
160
161    // Make sure the QuotaObserverChore has time to reflect the new region size reports
162    // (we saw above). The usage of the table should *not* decrease when we check it below,
163    // though, because the snapshot on our table will cause the table to "retain" the size.
164    TEST_UTIL.waitFor(20 * 1000, 500, new SpaceQuotaSnapshotPredicate(conn, tn) {
165      @Override
166      public boolean evaluate(SpaceQuotaSnapshot snapshot) throws Exception {
167        return snapshot.getUsage() >= finalSize;
168      }
169    });
170
171    // The final usage should be the sum of the initial size (referenced by the snapshot) and the
172    // new size we just wrote above.
173    long expectedFinalSize = actualInitialSize + finalSize;
174    LOG.info(
175        "Expecting table usage to be " + actualInitialSize + " + " + finalSize
176        + " = " + expectedFinalSize);
177    // The size of the table (WRT quotas) should now be approximately double what it was previously
178    TEST_UTIL.waitFor(30 * 1000, 1000, new SpaceQuotaSnapshotPredicate(conn, tn) {
179      @Override boolean evaluate(SpaceQuotaSnapshot snapshot) throws Exception {
180        LOG.debug("Checking for " + expectedFinalSize + " == " + snapshot.getUsage());
181        return expectedFinalSize == snapshot.getUsage();
182      }
183    });
184
185    Map<String,Long> snapshotSizes = QuotaTableUtil.getObservedSnapshotSizes(conn);
186    Long size = snapshotSizes.get(snapshot1);
187    assertNotNull("Did not observe the size of the snapshot", size);
188    assertEquals(
189        "The recorded size of the HBase snapshot was not the size we expected", actualInitialSize,
190        size.longValue());
191  }
192
193  @Test
194  public void testNamespacesInheritSnapshotSize() throws Exception {
195    String ns = helper.createNamespace().getName();
196    TableName tn = helper.createTableWithRegions(ns, 1);
197    LOG.info("Writing data");
198    // Set a quota
199    QuotaSettings settings = QuotaSettingsFactory.limitNamespaceSpace(
200        ns, SpaceQuotaHelperForTests.ONE_GIGABYTE, SpaceViolationPolicy.NO_INSERTS);
201    admin.setQuota(settings);
202
203    // Write some data and flush it to disk
204    final long initialSize = 2L * SpaceQuotaHelperForTests.ONE_MEGABYTE;
205    helper.writeData(tn, initialSize);
206    admin.flush(tn);
207
208    LOG.info("Waiting until namespace size reflects written data");
209    // Wait until that data is seen by the master
210    TEST_UTIL.waitFor(30 * 1000, 500, new SpaceQuotaSnapshotPredicate(conn, ns) {
211      @Override boolean evaluate(SpaceQuotaSnapshot snapshot) throws Exception {
212        return snapshot.getUsage() >= initialSize;
213      }
214    });
215
216    // Make sure we see the "final" new size for the table, not some intermediate
217    waitForStableQuotaSize(conn, null, ns);
218
219    // The actual size on disk after we wrote our data the first time
220    final long actualInitialSize = QuotaTableUtil.getCurrentSnapshot(conn, ns).getUsage();
221    LOG.info("Initial table size was " + actualInitialSize);
222
223    LOG.info("Snapshot the table");
224    final String snapshot1 = tn.getQualifierAsString() + "_snapshot1";
225    admin.snapshot(snapshot1, tn);
226
227    // Write the same data again, then flush+compact. This should make sure that
228    // the snapshot is referencing files that the table no longer references.
229    LOG.info("Write more data");
230    helper.writeData(tn, initialSize);
231    LOG.info("Flush the table");
232    admin.flush(tn);
233    LOG.info("Synchronously compacting the table");
234    TEST_UTIL.compact(tn, true);
235
236    final long upperBound = initialSize + FUDGE_FOR_TABLE_SIZE;
237    final long lowerBound = initialSize - FUDGE_FOR_TABLE_SIZE;
238
239    LOG.info("Waiting for the region reports to reflect the correct size, between ("
240        + lowerBound + ", " + upperBound + ")");
241    TEST_UTIL.waitFor(30 * 1000, 500, new Predicate<Exception>() {
242      @Override
243      public boolean evaluate() throws Exception {
244        Map<TableName,Long> sizes = QuotaTableUtil.getMasterReportedTableSizes(conn);
245        LOG.debug("Master observed table sizes from region size reports: " + sizes);
246        Long size = sizes.get(tn);
247        if (null == size) {
248          return false;
249        }
250        return size < upperBound && size > lowerBound;
251      }
252    });
253
254    // Make sure we see the "final" new size for the table, not some intermediate
255    waitForStableRegionSizeReport(conn, tn);
256    final long finalSize = getRegionSizeReportForTable(conn, tn);
257    assertNotNull("Did not expect to see a null size", finalSize);
258    LOG.info("Final observed size of table: " + finalSize);
259
260    // Make sure the QuotaObserverChore has time to reflect the new region size reports
261    // (we saw above). The usage of the table should *not* decrease when we check it below,
262    // though, because the snapshot on our table will cause the table to "retain" the size.
263    TEST_UTIL.waitFor(20 * 1000, 500, new SpaceQuotaSnapshotPredicate(conn, ns) {
264      @Override
265      public boolean evaluate(SpaceQuotaSnapshot snapshot) throws Exception {
266        return snapshot.getUsage() >= finalSize;
267      }
268    });
269
270    // The final usage should be the sum of the initial size (referenced by the snapshot) and the
271    // new size we just wrote above.
272    long expectedFinalSize = actualInitialSize + finalSize;
273    LOG.info(
274        "Expecting namespace usage to be " + actualInitialSize + " + " + finalSize
275        + " = " + expectedFinalSize);
276    // The size of the table (WRT quotas) should now be approximately double what it was previously
277    TEST_UTIL.waitFor(30 * 1000, 1000, new SpaceQuotaSnapshotPredicate(conn, ns) {
278      @Override boolean evaluate(SpaceQuotaSnapshot snapshot) throws Exception {
279        LOG.debug("Checking for " + expectedFinalSize + " == " + snapshot.getUsage());
280        return expectedFinalSize == snapshot.getUsage();
281      }
282    });
283
284    Map<String,Long> snapshotSizes = QuotaTableUtil.getObservedSnapshotSizes(conn);
285    Long size = snapshotSizes.get(snapshot1);
286    assertNotNull("Did not observe the size of the snapshot", size);
287    assertEquals(
288        "The recorded size of the HBase snapshot was not the size we expected", actualInitialSize,
289        size.longValue());
290  }
291
292  @Test
293  public void testTablesWithSnapshots() throws Exception {
294    final Connection conn = TEST_UTIL.getConnection();
295    final SpaceViolationPolicy policy = SpaceViolationPolicy.NO_INSERTS;
296    final TableName tn = helper.createTableWithRegions(10);
297
298    // 3MB limit on the table
299    final long tableLimit = 3L * SpaceQuotaHelperForTests.ONE_MEGABYTE;
300    TEST_UTIL.getAdmin().setQuota(QuotaSettingsFactory.limitTableSpace(tn, tableLimit, policy));
301
302    LOG.info("Writing first data set");
303    // Write more data than should be allowed and flush it to disk
304    helper.writeData(tn, 1L * SpaceQuotaHelperForTests.ONE_MEGABYTE, "q1");
305
306    LOG.info("Creating snapshot");
307    TEST_UTIL.getAdmin().snapshot(tn.toString() + "snap1", tn, SnapshotType.FLUSH);
308
309    LOG.info("Writing second data set");
310    // Write some more data
311    helper.writeData(tn, 1L * SpaceQuotaHelperForTests.ONE_MEGABYTE, "q2");
312
313    LOG.info("Flushing and major compacting table");
314    // Compact the table to force the snapshot to own all of its files
315    TEST_UTIL.getAdmin().flush(tn);
316    TEST_UTIL.compact(tn, true);
317
318    LOG.info("Checking for quota violation");
319    // Wait to observe the quota moving into violation
320    TEST_UTIL.waitFor(60_000, 1_000, new Predicate<Exception>() {
321      @Override
322      public boolean evaluate() throws Exception {
323        Scan s = QuotaTableUtil.makeQuotaSnapshotScanForTable(tn);
324        try (Table t = conn.getTable(QuotaTableUtil.QUOTA_TABLE_NAME)) {
325          ResultScanner rs = t.getScanner(s);
326          try {
327            Result r = Iterables.getOnlyElement(rs);
328            CellScanner cs = r.cellScanner();
329            assertTrue(cs.advance());
330            Cell c = cs.current();
331            SpaceQuotaSnapshot snapshot = SpaceQuotaSnapshot.toSpaceQuotaSnapshot(
332                QuotaProtos.SpaceQuotaSnapshot.parseFrom(
333                  UnsafeByteOperations.unsafeWrap(
334                      c.getValueArray(), c.getValueOffset(), c.getValueLength())));
335            LOG.info(
336                snapshot.getUsage() + "/" + snapshot.getLimit() + " " + snapshot.getQuotaStatus());
337            // We expect to see the table move to violation
338            return snapshot.getQuotaStatus().isInViolation();
339          } finally {
340            if (null != rs) {
341              rs.close();
342            }
343          }
344        }
345      }
346    });
347  }
348
349  @Test
350  public void testRematerializedTablesDoNoInheritSpace() throws Exception {
351    TableName tn = helper.createTableWithRegions(1);
352    TableName tn2 = helper.getNextTableName();
353    LOG.info("Writing data");
354    // Set a quota on both tables
355    QuotaSettings settings = QuotaSettingsFactory.limitTableSpace(
356        tn, SpaceQuotaHelperForTests.ONE_GIGABYTE, SpaceViolationPolicy.NO_INSERTS);
357    admin.setQuota(settings);
358    QuotaSettings settings2 = QuotaSettingsFactory.limitTableSpace(
359        tn2, SpaceQuotaHelperForTests.ONE_GIGABYTE, SpaceViolationPolicy.NO_INSERTS);
360    admin.setQuota(settings2);
361    // Write some data
362    final long initialSize = 2L * SpaceQuotaHelperForTests.ONE_MEGABYTE;
363    helper.writeData(tn, initialSize);
364
365    LOG.info("Waiting until table size reflects written data");
366    // Wait until that data is seen by the master
367    TEST_UTIL.waitFor(30 * 1000, 500, new SpaceQuotaSnapshotPredicate(conn, tn) {
368      @Override boolean evaluate(SpaceQuotaSnapshot snapshot) throws Exception {
369        return snapshot.getUsage() >= initialSize;
370      }
371    });
372
373    // Make sure we see the final quota usage size
374    waitForStableQuotaSize(conn, tn, null);
375
376    // The actual size on disk after we wrote our data the first time
377    final long actualInitialSize = QuotaTableUtil.getCurrentSnapshot(conn, tn).getUsage();
378    LOG.info("Initial table size was " + actualInitialSize);
379
380    LOG.info("Snapshot the table");
381    final String snapshot1 = tn.toString() + "_snapshot1";
382    admin.snapshot(snapshot1, tn);
383
384    admin.cloneSnapshot(snapshot1, tn2);
385
386    // Write some more data to the first table
387    helper.writeData(tn, initialSize, "q2");
388    admin.flush(tn);
389
390    // Watch the usage of the first table with some more data to know when the new
391    // region size reports were sent to the master
392    TEST_UTIL.waitFor(30_000, 1_000, new SpaceQuotaSnapshotPredicate(conn, tn) {
393      @Override
394      boolean evaluate(SpaceQuotaSnapshot snapshot) throws Exception {
395        return snapshot.getUsage() >= actualInitialSize * 2;
396      }
397    });
398
399    // We know that reports were sent by our RS, verify that they take up zero size.
400    SpaceQuotaSnapshot snapshot = QuotaTableUtil.getCurrentSnapshot(conn, tn2);
401    assertNotNull(snapshot);
402    assertEquals(0, snapshot.getUsage());
403
404    // Compact the cloned table to force it to own its own files.
405    TEST_UTIL.compact(tn2, true);
406    // After the table is compacted, it should have its own files and be the same size as originally
407    TEST_UTIL.waitFor(30_000, 1_000, new SpaceQuotaSnapshotPredicate(conn, tn2) {
408      @Override
409      boolean evaluate(SpaceQuotaSnapshot snapshot) throws Exception {
410        return snapshot.getUsage() == actualInitialSize;
411      }
412    });
413  }
414
415  void waitForStableQuotaSize(Connection conn, TableName tn, String ns) throws Exception {
416    // For some stability in the value before proceeding
417    // Helps make sure that we got the actual last value, not some inbetween
418    AtomicLong lastValue = new AtomicLong(-1);
419    AtomicInteger counter = new AtomicInteger(0);
420    TEST_UTIL.waitFor(15_000, 500, new SpaceQuotaSnapshotPredicate(conn, tn, ns) {
421      @Override boolean evaluate(SpaceQuotaSnapshot snapshot) throws Exception {
422        LOG.debug("Last observed size=" + lastValue.get());
423        if (snapshot.getUsage() == lastValue.get()) {
424          int numMatches = counter.incrementAndGet();
425          if (numMatches >= 5) {
426            return true;
427          }
428          // Not yet..
429          return false;
430        }
431        counter.set(0);
432        lastValue.set(snapshot.getUsage());
433        return false;
434      }
435    });
436  }
437
438  long getRegionSizeReportForTable(Connection conn, TableName tn) throws IOException {
439    Map<TableName,Long> sizes = QuotaTableUtil.getMasterReportedTableSizes(conn);
440    Long value = sizes.get(tn);
441    if (null == value) {
442      return 0L;
443    }
444    return value.longValue();
445  }
446
447  void waitForStableRegionSizeReport(Connection conn, TableName tn) throws Exception {
448    // For some stability in the value before proceeding
449    // Helps make sure that we got the actual last value, not some inbetween
450    AtomicLong lastValue = new AtomicLong(-1);
451    AtomicInteger counter = new AtomicInteger(0);
452    TEST_UTIL.waitFor(15_000, 500, new Predicate<Exception>() {
453      @Override public boolean evaluate() throws Exception {
454        LOG.debug("Last observed size=" + lastValue.get());
455        long actual = getRegionSizeReportForTable(conn, tn);
456        if (actual == lastValue.get()) {
457          int numMatches = counter.incrementAndGet();
458          if (numMatches >= 5) {
459            return true;
460          }
461          // Not yet..
462          return false;
463        }
464        counter.set(0);
465        lastValue.set(actual);
466        return false;
467      }
468    });
469  }
470}