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.master.cleaner;
019
020import static org.junit.Assert.assertEquals;
021import static org.junit.Assert.assertFalse;
022import static org.junit.Assert.assertTrue;
023
024import java.io.IOException;
025import java.util.List;
026import java.util.Random;
027import java.util.concurrent.ThreadLocalRandom;
028import org.apache.hadoop.conf.Configuration;
029import org.apache.hadoop.fs.FSDataOutputStream;
030import org.apache.hadoop.fs.FileStatus;
031import org.apache.hadoop.fs.FileSystem;
032import org.apache.hadoop.fs.Path;
033import org.apache.hadoop.hbase.ChoreService;
034import org.apache.hadoop.hbase.CoordinatedStateManager;
035import org.apache.hadoop.hbase.HBaseClassTestRule;
036import org.apache.hadoop.hbase.HBaseTestingUtility;
037import org.apache.hadoop.hbase.HConstants;
038import org.apache.hadoop.hbase.HRegionInfo;
039import org.apache.hadoop.hbase.Server;
040import org.apache.hadoop.hbase.ServerName;
041import org.apache.hadoop.hbase.TableName;
042import org.apache.hadoop.hbase.client.ClusterConnection;
043import org.apache.hadoop.hbase.client.Connection;
044import org.apache.hadoop.hbase.mob.ManualMobMaintHFileCleaner;
045import org.apache.hadoop.hbase.mob.MobUtils;
046import org.apache.hadoop.hbase.testclassification.MasterTests;
047import org.apache.hadoop.hbase.testclassification.MediumTests;
048import org.apache.hadoop.hbase.util.EnvironmentEdge;
049import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
050import org.apache.hadoop.hbase.util.HFileArchiveUtil;
051import org.apache.hadoop.hbase.zookeeper.ZKWatcher;
052import org.junit.AfterClass;
053import org.junit.Assert;
054import org.junit.BeforeClass;
055import org.junit.ClassRule;
056import org.junit.Test;
057import org.junit.experimental.categories.Category;
058import org.slf4j.Logger;
059import org.slf4j.LoggerFactory;
060
061@Category({ MasterTests.class, MediumTests.class })
062public class TestHFileCleaner {
063
064  @ClassRule
065  public static final HBaseClassTestRule CLASS_RULE =
066    HBaseClassTestRule.forClass(TestHFileCleaner.class);
067
068  private static final Logger LOG = LoggerFactory.getLogger(TestHFileCleaner.class);
069
070  private final static HBaseTestingUtility UTIL = new HBaseTestingUtility();
071
072  private static DirScanPool POOL;
073
074  @BeforeClass
075  public static void setupCluster() throws Exception {
076    // have to use a minidfs cluster because the localfs doesn't modify file times correctly
077    UTIL.startMiniDFSCluster(1);
078    POOL = DirScanPool.getHFileCleanerScanPool(UTIL.getConfiguration());
079  }
080
081  @AfterClass
082  public static void shutdownCluster() throws IOException {
083    UTIL.shutdownMiniDFSCluster();
084    POOL.shutdownNow();
085  }
086
087  @Test
088  public void testTTLCleaner() throws IOException, InterruptedException {
089    FileSystem fs = UTIL.getDFSCluster().getFileSystem();
090    Path root = UTIL.getDataTestDirOnTestFS();
091    Path file = new Path(root, "file");
092    fs.createNewFile(file);
093    long createTime = EnvironmentEdgeManager.currentTime();
094    assertTrue("Test file not created!", fs.exists(file));
095    TimeToLiveHFileCleaner cleaner = new TimeToLiveHFileCleaner();
096    // update the time info for the file, so the cleaner removes it
097    fs.setTimes(file, createTime - 100, -1);
098    Configuration conf = UTIL.getConfiguration();
099    conf.setLong(TimeToLiveHFileCleaner.TTL_CONF_KEY, 100);
100    cleaner.setConf(conf);
101    assertTrue("File not set deletable - check mod time:" + getFileStats(file, fs)
102      + " with create time:" + createTime, cleaner.isFileDeletable(fs.getFileStatus(file)));
103  }
104
105  @Test
106  public void testManualMobCleanerStopsMobRemoval() throws IOException {
107    FileSystem fs = UTIL.getDFSCluster().getFileSystem();
108    Path root = UTIL.getDataTestDirOnTestFS();
109    TableName table = TableName.valueOf("testManualMobCleanerStopsMobRemoval");
110    Path mob = HFileArchiveUtil.getRegionArchiveDir(root, table,
111      MobUtils.getMobRegionInfo(table).getEncodedName());
112    Path family = new Path(mob, "family");
113
114    Path file = new Path(family, "someHFileThatWouldBeAUUID");
115    fs.createNewFile(file);
116    assertTrue("Test file not created!", fs.exists(file));
117
118    ManualMobMaintHFileCleaner cleaner = new ManualMobMaintHFileCleaner();
119
120    assertFalse("Mob File shouldn't have been deletable. check path. '" + file + "'",
121      cleaner.isFileDeletable(fs.getFileStatus(file)));
122  }
123
124  @Test
125  public void testManualMobCleanerLetsNonMobGo() throws IOException {
126    FileSystem fs = UTIL.getDFSCluster().getFileSystem();
127    Path root = UTIL.getDataTestDirOnTestFS();
128    TableName table = TableName.valueOf("testManualMobCleanerLetsNonMobGo");
129    Path nonmob =
130      HFileArchiveUtil.getRegionArchiveDir(root, table, new HRegionInfo(table).getEncodedName());
131    Path family = new Path(nonmob, "family");
132
133    Path file = new Path(family, "someHFileThatWouldBeAUUID");
134    fs.createNewFile(file);
135    assertTrue("Test file not created!", fs.exists(file));
136
137    ManualMobMaintHFileCleaner cleaner = new ManualMobMaintHFileCleaner();
138
139    assertTrue("Non-Mob File should have been deletable. check path. '" + file + "'",
140      cleaner.isFileDeletable(fs.getFileStatus(file)));
141  }
142
143  /**
144   * @param file to check
145   * @return loggable information about the file
146   */
147  private String getFileStats(Path file, FileSystem fs) throws IOException {
148    FileStatus status = fs.getFileStatus(file);
149    return "File" + file + ", mtime:" + status.getModificationTime() + ", atime:"
150      + status.getAccessTime();
151  }
152
153  @Test
154  public void testHFileCleaning() throws Exception {
155    final EnvironmentEdge originalEdge = EnvironmentEdgeManager.getDelegate();
156    String prefix = "someHFileThatWouldBeAUUID";
157    Configuration conf = UTIL.getConfiguration();
158    // set TTL
159    long ttl = 2000;
160    conf.set(HFileCleaner.MASTER_HFILE_CLEANER_PLUGINS,
161      "org.apache.hadoop.hbase.master.cleaner.TimeToLiveHFileCleaner,"
162        + "org.apache.hadoop.hbase.mob.ManualMobMaintHFileCleaner");
163    conf.setLong(TimeToLiveHFileCleaner.TTL_CONF_KEY, ttl);
164    Server server = new DummyServer();
165    Path archivedHfileDir =
166      new Path(UTIL.getDataTestDirOnTestFS(), HConstants.HFILE_ARCHIVE_DIRECTORY);
167    FileSystem fs = FileSystem.get(conf);
168    HFileCleaner cleaner = new HFileCleaner(1000, server, conf, fs, archivedHfileDir, POOL);
169
170    // Create 2 invalid files, 1 "recent" file, 1 very new file and 30 old files
171    final long createTime = EnvironmentEdgeManager.currentTime();
172    fs.delete(archivedHfileDir, true);
173    fs.mkdirs(archivedHfileDir);
174    // Case 1: 1 invalid file, which should be deleted directly
175    fs.createNewFile(new Path(archivedHfileDir, "dfd-dfd"));
176    // Case 2: 1 "recent" file, not even deletable for the first log cleaner
177    // (TimeToLiveLogCleaner), so we are not going down the chain
178    LOG.debug("Now is: " + createTime);
179    for (int i = 1; i < 32; i++) {
180      // Case 3: old files which would be deletable for the first log cleaner
181      // (TimeToLiveHFileCleaner),
182      Path fileName = new Path(archivedHfileDir, (prefix + "." + (createTime + i)));
183      fs.createNewFile(fileName);
184      // set the creation time past ttl to ensure that it gets removed
185      fs.setTimes(fileName, createTime - ttl - 1, -1);
186      LOG.debug("Creating " + getFileStats(fileName, fs));
187    }
188
189    // Case 2: 1 newer file, not even deletable for the first log cleaner
190    // (TimeToLiveLogCleaner), so we are not going down the chain
191    Path saved = new Path(archivedHfileDir, prefix + ".00000000000");
192    fs.createNewFile(saved);
193    // set creation time within the ttl
194    fs.setTimes(saved, createTime - ttl / 2, -1);
195    LOG.debug("Creating " + getFileStats(saved, fs));
196    for (FileStatus stat : fs.listStatus(archivedHfileDir)) {
197      LOG.debug(stat.getPath().toString());
198    }
199
200    assertEquals(33, fs.listStatus(archivedHfileDir).length);
201
202    // set a custom edge manager to handle time checking
203    EnvironmentEdge setTime = new EnvironmentEdge() {
204      @Override
205      public long currentTime() {
206        return createTime;
207      }
208    };
209    EnvironmentEdgeManager.injectEdge(setTime);
210
211    // run the chore
212    cleaner.chore();
213
214    // ensure we only end up with the saved file
215    assertEquals(1, fs.listStatus(archivedHfileDir).length);
216
217    for (FileStatus file : fs.listStatus(archivedHfileDir)) {
218      LOG.debug("Kept hfiles: " + file.getPath().getName());
219    }
220
221    // reset the edge back to the original edge
222    EnvironmentEdgeManager.injectEdge(originalEdge);
223  }
224
225  @Test
226  public void testRemovesEmptyDirectories() throws Exception {
227    Configuration conf = UTIL.getConfiguration();
228    // no cleaner policies = delete all files
229    conf.setStrings(HFileCleaner.MASTER_HFILE_CLEANER_PLUGINS, "");
230    Server server = new DummyServer();
231    Path archivedHfileDir =
232      new Path(UTIL.getDataTestDirOnTestFS(), HConstants.HFILE_ARCHIVE_DIRECTORY);
233
234    // setup the cleaner
235    FileSystem fs = UTIL.getDFSCluster().getFileSystem();
236    HFileCleaner cleaner = new HFileCleaner(1000, server, conf, fs, archivedHfileDir, POOL);
237
238    // make all the directories for archiving files
239    Path table = new Path(archivedHfileDir, "table");
240    Path region = new Path(table, "regionsomthing");
241    Path family = new Path(region, "fam");
242    Path file = new Path(family, "file12345");
243    fs.mkdirs(family);
244    if (!fs.exists(family)) throw new RuntimeException("Couldn't create test family:" + family);
245    fs.create(file).close();
246    if (!fs.exists(file)) throw new RuntimeException("Test file didn't get created:" + file);
247
248    // run the chore to cleanup the files (and the directories above it)
249    cleaner.chore();
250
251    // make sure all the parent directories get removed
252    assertFalse("family directory not removed for empty directory", fs.exists(family));
253    assertFalse("region directory not removed for empty directory", fs.exists(region));
254    assertFalse("table directory not removed for empty directory", fs.exists(table));
255    assertTrue("archive directory", fs.exists(archivedHfileDir));
256  }
257
258  static class DummyServer implements Server {
259    @Override
260    public Configuration getConfiguration() {
261      return UTIL.getConfiguration();
262    }
263
264    @Override
265    public ZKWatcher getZooKeeper() {
266      try {
267        return new ZKWatcher(getConfiguration(), "dummy server", this);
268      } catch (IOException e) {
269        e.printStackTrace();
270      }
271      return null;
272    }
273
274    @Override
275    public CoordinatedStateManager getCoordinatedStateManager() {
276      return null;
277    }
278
279    @Override
280    public ClusterConnection getConnection() {
281      return null;
282    }
283
284    @Override
285    public ServerName getServerName() {
286      return ServerName.valueOf("regionserver,60020,000000");
287    }
288
289    @Override
290    public void abort(String why, Throwable e) {
291    }
292
293    @Override
294    public boolean isAborted() {
295      return false;
296    }
297
298    @Override
299    public void stop(String why) {
300    }
301
302    @Override
303    public boolean isStopped() {
304      return false;
305    }
306
307    @Override
308    public ChoreService getChoreService() {
309      return null;
310    }
311
312    @Override
313    public ClusterConnection getClusterConnection() {
314      // TODO Auto-generated method stub
315      return null;
316    }
317
318    @Override
319    public FileSystem getFileSystem() {
320      return null;
321    }
322
323    @Override
324    public boolean isStopping() {
325      return false;
326    }
327
328    @Override
329    public Connection createConnection(Configuration conf) throws IOException {
330      return null;
331    }
332  }
333
334  @Test
335  public void testThreadCleanup() throws Exception {
336    Configuration conf = UTIL.getConfiguration();
337    conf.setStrings(HFileCleaner.MASTER_HFILE_CLEANER_PLUGINS, "");
338    Server server = new DummyServer();
339    Path archivedHfileDir =
340      new Path(UTIL.getDataTestDirOnTestFS(), HConstants.HFILE_ARCHIVE_DIRECTORY);
341
342    // setup the cleaner
343    FileSystem fs = UTIL.getDFSCluster().getFileSystem();
344    HFileCleaner cleaner = new HFileCleaner(1000, server, conf, fs, archivedHfileDir, POOL);
345    // clean up archive directory
346    fs.delete(archivedHfileDir, true);
347    fs.mkdirs(archivedHfileDir);
348    // create some file to delete
349    fs.createNewFile(new Path(archivedHfileDir, "dfd-dfd"));
350    // launch the chore
351    cleaner.chore();
352    // call cleanup
353    cleaner.cleanup();
354    // wait awhile for thread to die
355    Thread.sleep(100);
356    for (Thread thread : cleaner.getCleanerThreads()) {
357      Assert.assertFalse(thread.isAlive());
358    }
359  }
360
361  @Test
362  public void testLargeSmallIsolation() throws Exception {
363    Configuration conf = UTIL.getConfiguration();
364    // no cleaner policies = delete all files
365    conf.setStrings(HFileCleaner.MASTER_HFILE_CLEANER_PLUGINS, "");
366    conf.setInt(HFileCleaner.HFILE_DELETE_THROTTLE_THRESHOLD, 512 * 1024);
367    Server server = new DummyServer();
368    Path archivedHfileDir =
369      new Path(UTIL.getDataTestDirOnTestFS(), HConstants.HFILE_ARCHIVE_DIRECTORY);
370
371    // setup the cleaner
372    FileSystem fs = UTIL.getDFSCluster().getFileSystem();
373    HFileCleaner cleaner = new HFileCleaner(1000, server, conf, fs, archivedHfileDir, POOL);
374    // clean up archive directory
375    fs.delete(archivedHfileDir, true);
376    fs.mkdirs(archivedHfileDir);
377    // necessary set up
378    final int LARGE_FILE_NUM = 5;
379    final int SMALL_FILE_NUM = 20;
380    createFilesForTesting(LARGE_FILE_NUM, SMALL_FILE_NUM, fs, archivedHfileDir);
381    // call cleanup
382    cleaner.chore();
383
384    Assert.assertEquals(LARGE_FILE_NUM, cleaner.getNumOfDeletedLargeFiles());
385    Assert.assertEquals(SMALL_FILE_NUM, cleaner.getNumOfDeletedSmallFiles());
386  }
387
388  @Test
389  public void testOnConfigurationChange() throws Exception {
390    // constants
391    final int ORIGINAL_THROTTLE_POINT = 512 * 1024;
392    final int ORIGINAL_QUEUE_INIT_SIZE = 512;
393    final int UPDATE_THROTTLE_POINT = 1024;// small enough to change large/small check
394    final int UPDATE_QUEUE_INIT_SIZE = 1024;
395    final int LARGE_FILE_NUM = 5;
396    final int SMALL_FILE_NUM = 20;
397    final int LARGE_THREAD_NUM = 2;
398    final int SMALL_THREAD_NUM = 4;
399    final long THREAD_TIMEOUT_MSEC = 30 * 1000L;
400    final long THREAD_CHECK_INTERVAL_MSEC = 500L;
401
402    Configuration conf = UTIL.getConfiguration();
403    // no cleaner policies = delete all files
404    conf.setStrings(HFileCleaner.MASTER_HFILE_CLEANER_PLUGINS, "");
405    conf.setInt(HFileCleaner.HFILE_DELETE_THROTTLE_THRESHOLD, ORIGINAL_THROTTLE_POINT);
406    conf.setInt(HFileCleaner.LARGE_HFILE_QUEUE_INIT_SIZE, ORIGINAL_QUEUE_INIT_SIZE);
407    conf.setInt(HFileCleaner.SMALL_HFILE_QUEUE_INIT_SIZE, ORIGINAL_QUEUE_INIT_SIZE);
408    Server server = new DummyServer();
409    Path archivedHfileDir =
410      new Path(UTIL.getDataTestDirOnTestFS(), HConstants.HFILE_ARCHIVE_DIRECTORY);
411
412    // setup the cleaner
413    FileSystem fs = UTIL.getDFSCluster().getFileSystem();
414    final HFileCleaner cleaner = new HFileCleaner(1000, server, conf, fs, archivedHfileDir, POOL);
415    Assert.assertEquals(ORIGINAL_THROTTLE_POINT, cleaner.getThrottlePoint());
416    Assert.assertEquals(ORIGINAL_QUEUE_INIT_SIZE, cleaner.getLargeQueueInitSize());
417    Assert.assertEquals(ORIGINAL_QUEUE_INIT_SIZE, cleaner.getSmallQueueInitSize());
418    Assert.assertEquals(HFileCleaner.DEFAULT_HFILE_DELETE_THREAD_TIMEOUT_MSEC,
419      cleaner.getCleanerThreadTimeoutMsec());
420    Assert.assertEquals(HFileCleaner.DEFAULT_HFILE_DELETE_THREAD_CHECK_INTERVAL_MSEC,
421      cleaner.getCleanerThreadCheckIntervalMsec());
422
423    // clean up archive directory and create files for testing
424    fs.delete(archivedHfileDir, true);
425    fs.mkdirs(archivedHfileDir);
426    createFilesForTesting(LARGE_FILE_NUM, SMALL_FILE_NUM, fs, archivedHfileDir);
427
428    // call cleaner, run as daemon to test the interrupt-at-middle case
429    Thread t = new Thread() {
430      @Override
431      public void run() {
432        cleaner.chore();
433      }
434    };
435    t.setDaemon(true);
436    t.start();
437    // wait until file clean started
438    while (cleaner.getNumOfDeletedSmallFiles() == 0) {
439      Thread.yield();
440    }
441
442    // trigger configuration change
443    Configuration newConf = new Configuration(conf);
444    newConf.setInt(HFileCleaner.HFILE_DELETE_THROTTLE_THRESHOLD, UPDATE_THROTTLE_POINT);
445    newConf.setInt(HFileCleaner.LARGE_HFILE_QUEUE_INIT_SIZE, UPDATE_QUEUE_INIT_SIZE);
446    newConf.setInt(HFileCleaner.SMALL_HFILE_QUEUE_INIT_SIZE, UPDATE_QUEUE_INIT_SIZE);
447    newConf.setInt(HFileCleaner.LARGE_HFILE_DELETE_THREAD_NUMBER, LARGE_THREAD_NUM);
448    newConf.setInt(HFileCleaner.SMALL_HFILE_DELETE_THREAD_NUMBER, SMALL_THREAD_NUM);
449    newConf.setLong(HFileCleaner.HFILE_DELETE_THREAD_TIMEOUT_MSEC, THREAD_TIMEOUT_MSEC);
450    newConf.setLong(HFileCleaner.HFILE_DELETE_THREAD_CHECK_INTERVAL_MSEC,
451      THREAD_CHECK_INTERVAL_MSEC);
452
453    LOG.debug("File deleted from large queue: " + cleaner.getNumOfDeletedLargeFiles()
454      + "; from small queue: " + cleaner.getNumOfDeletedSmallFiles());
455    cleaner.onConfigurationChange(newConf);
456
457    // check values after change
458    Assert.assertEquals(UPDATE_THROTTLE_POINT, cleaner.getThrottlePoint());
459    Assert.assertEquals(UPDATE_QUEUE_INIT_SIZE, cleaner.getLargeQueueInitSize());
460    Assert.assertEquals(UPDATE_QUEUE_INIT_SIZE, cleaner.getSmallQueueInitSize());
461    Assert.assertEquals(LARGE_THREAD_NUM + SMALL_THREAD_NUM, cleaner.getCleanerThreads().size());
462    Assert.assertEquals(THREAD_TIMEOUT_MSEC, cleaner.getCleanerThreadTimeoutMsec());
463    Assert.assertEquals(THREAD_CHECK_INTERVAL_MSEC, cleaner.getCleanerThreadCheckIntervalMsec());
464
465    // make sure no cost when onConfigurationChange called with no change
466    List<Thread> oldThreads = cleaner.getCleanerThreads();
467    cleaner.onConfigurationChange(newConf);
468    List<Thread> newThreads = cleaner.getCleanerThreads();
469    Assert.assertArrayEquals(oldThreads.toArray(), newThreads.toArray());
470
471    // wait until clean done and check
472    t.join();
473    LOG.debug("File deleted from large queue: " + cleaner.getNumOfDeletedLargeFiles()
474      + "; from small queue: " + cleaner.getNumOfDeletedSmallFiles());
475    Assert.assertTrue(
476      "Should delete more than " + LARGE_FILE_NUM + " files from large queue but actually "
477        + cleaner.getNumOfDeletedLargeFiles(),
478      cleaner.getNumOfDeletedLargeFiles() > LARGE_FILE_NUM);
479    Assert.assertTrue(
480      "Should delete less than " + SMALL_FILE_NUM + " files from small queue but actually "
481        + cleaner.getNumOfDeletedSmallFiles(),
482      cleaner.getNumOfDeletedSmallFiles() < SMALL_FILE_NUM);
483  }
484
485  private void createFilesForTesting(int largeFileNum, int smallFileNum, FileSystem fs,
486    Path archivedHfileDir) throws IOException {
487    final Random rand = ThreadLocalRandom.current();
488    final byte[] large = new byte[1024 * 1024];
489    for (int i = 0; i < large.length; i++) {
490      large[i] = (byte) rand.nextInt(128);
491    }
492    final byte[] small = new byte[1024];
493    for (int i = 0; i < small.length; i++) {
494      small[i] = (byte) rand.nextInt(128);
495    }
496    // create large and small files
497    for (int i = 1; i <= largeFileNum; i++) {
498      FSDataOutputStream out = fs.create(new Path(archivedHfileDir, "large-file-" + i));
499      out.write(large);
500      out.close();
501    }
502    for (int i = 1; i <= smallFileNum; i++) {
503      FSDataOutputStream out = fs.create(new Path(archivedHfileDir, "small-file-" + i));
504      out.write(small);
505      out.close();
506    }
507  }
508}