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