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.backup;
019
020import java.io.FileNotFoundException;
021import java.io.IOException;
022import java.io.InterruptedIOException;
023import java.util.ArrayList;
024import java.util.Collection;
025import java.util.Collections;
026import java.util.List;
027import java.util.concurrent.ExecutionException;
028import java.util.concurrent.Future;
029import java.util.concurrent.ThreadFactory;
030import java.util.concurrent.ThreadPoolExecutor;
031import java.util.concurrent.TimeUnit;
032import java.util.concurrent.atomic.AtomicInteger;
033import java.util.function.Function;
034import java.util.stream.Collectors;
035import java.util.stream.Stream;
036import org.apache.hadoop.conf.Configuration;
037import org.apache.hadoop.fs.FileStatus;
038import org.apache.hadoop.fs.FileSystem;
039import org.apache.hadoop.fs.Path;
040import org.apache.hadoop.fs.PathFilter;
041import org.apache.hadoop.hbase.HConstants;
042import org.apache.hadoop.hbase.client.RegionInfo;
043import org.apache.hadoop.hbase.regionserver.HStoreFile;
044import org.apache.hadoop.hbase.util.Bytes;
045import org.apache.hadoop.hbase.util.CommonFSUtils;
046import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
047import org.apache.hadoop.hbase.util.FSUtils;
048import org.apache.hadoop.hbase.util.HFileArchiveUtil;
049import org.apache.hadoop.hbase.util.Threads;
050import org.apache.hadoop.io.MultipleIOException;
051import org.apache.yetus.audience.InterfaceAudience;
052import org.slf4j.Logger;
053import org.slf4j.LoggerFactory;
054
055import org.apache.hbase.thirdparty.com.google.common.base.Preconditions;
056
057/**
058 * Utility class to handle the removal of HFiles (or the respective {@link HStoreFile StoreFiles})
059 * for a HRegion from the {@link FileSystem}. The hfiles will be archived or deleted, depending on
060 * the state of the system.
061 */
062@InterfaceAudience.Private
063public class HFileArchiver {
064  private static final Logger LOG = LoggerFactory.getLogger(HFileArchiver.class);
065  private static final String SEPARATOR = ".";
066
067  /** Number of retries in case of fs operation failure */
068  private static final int DEFAULT_RETRIES_NUMBER = 3;
069
070  private static final Function<File, Path> FUNC_FILE_TO_PATH = new Function<File, Path>() {
071    @Override
072    public Path apply(File file) {
073      return file == null ? null : file.getPath();
074    }
075  };
076
077  private static ThreadPoolExecutor archiveExecutor;
078
079  private HFileArchiver() {
080    // hidden ctor since this is just a util
081  }
082
083  /** Returns True if the Region exits in the filesystem. */
084  public static boolean exists(Configuration conf, FileSystem fs, RegionInfo info)
085    throws IOException {
086    Path rootDir = CommonFSUtils.getRootDir(conf);
087    Path regionDir = FSUtils.getRegionDirFromRootDir(rootDir, info);
088    return fs.exists(regionDir);
089  }
090
091  /**
092   * Cleans up all the files for a HRegion by archiving the HFiles to the archive directory
093   * @param conf the configuration to use
094   * @param fs   the file system object
095   * @param info RegionInfo for region to be deleted
096   */
097  public static void archiveRegion(Configuration conf, FileSystem fs, RegionInfo info)
098    throws IOException {
099    Path rootDir = CommonFSUtils.getRootDir(conf);
100    archiveRegion(fs, rootDir, CommonFSUtils.getTableDir(rootDir, info.getTable()),
101      FSUtils.getRegionDirFromRootDir(rootDir, info));
102  }
103
104  /**
105   * Remove an entire region from the table directory via archiving the region's hfiles.
106   * @param fs        {@link FileSystem} from which to remove the region
107   * @param rootdir   {@link Path} to the root directory where hbase files are stored (for building
108   *                  the archive path)
109   * @param tableDir  {@link Path} to where the table is being stored (for building the archive
110   *                  path)
111   * @param regionDir {@link Path} to where a region is being stored (for building the archive path)
112   * @return <tt>true</tt> if the region was successfully deleted. <tt>false</tt> if the filesystem
113   *         operations could not complete.
114   * @throws IOException if the request cannot be completed
115   */
116  public static boolean archiveRegion(FileSystem fs, Path rootdir, Path tableDir, Path regionDir)
117    throws IOException {
118    // otherwise, we archive the files
119    // make sure we can archive
120    if (tableDir == null || regionDir == null) {
121      LOG.error("No archive directory could be found because tabledir (" + tableDir
122        + ") or regiondir (" + regionDir + "was null. Deleting files instead.");
123      if (regionDir != null) {
124        deleteRegionWithoutArchiving(fs, regionDir);
125      }
126      // we should have archived, but failed to. Doesn't matter if we deleted
127      // the archived files correctly or not.
128      return false;
129    }
130
131    LOG.debug("ARCHIVING {}", regionDir);
132
133    // make sure the regiondir lives under the tabledir
134    Preconditions.checkArgument(regionDir.toString().startsWith(tableDir.toString()));
135    Path regionArchiveDir = HFileArchiveUtil.getRegionArchiveDir(rootdir,
136      CommonFSUtils.getTableName(tableDir), regionDir.getName());
137
138    FileStatusConverter getAsFile = new FileStatusConverter(fs);
139    // otherwise, we attempt to archive the store files
140
141    // build collection of just the store directories to archive
142    Collection<File> toArchive = new ArrayList<>();
143    final PathFilter dirFilter = new FSUtils.DirFilter(fs);
144    PathFilter nonHidden = new PathFilter() {
145      @Override
146      public boolean accept(Path file) {
147        return dirFilter.accept(file) && !file.getName().startsWith(".");
148      }
149    };
150    FileStatus[] storeDirs = CommonFSUtils.listStatus(fs, regionDir, nonHidden);
151    // if there no files, we can just delete the directory and return;
152    if (storeDirs == null) {
153      LOG.debug("Directory {} empty.", regionDir);
154      return deleteRegionWithoutArchiving(fs, regionDir);
155    }
156
157    // convert the files in the region to a File
158    Stream.of(storeDirs).map(getAsFile).forEachOrdered(toArchive::add);
159    LOG.debug("Archiving " + toArchive);
160    List<File> failedArchive =
161      resolveAndArchive(fs, regionArchiveDir, toArchive, EnvironmentEdgeManager.currentTime());
162    if (!failedArchive.isEmpty()) {
163      throw new FailedArchiveException(
164        "Failed to archive/delete all the files for region:" + regionDir.getName() + " into "
165          + regionArchiveDir + ". Something is probably awry on the filesystem.",
166        failedArchive.stream().map(FUNC_FILE_TO_PATH).collect(Collectors.toList()));
167    }
168    // if that was successful, then we delete the region
169    return deleteRegionWithoutArchiving(fs, regionDir);
170  }
171
172  /**
173   * Archive the specified regions in parallel.
174   * @param conf          the configuration to use
175   * @param fs            {@link FileSystem} from which to remove the region
176   * @param rootDir       {@link Path} to the root directory where hbase files are stored (for
177   *                      building the archive path)
178   * @param tableDir      {@link Path} to where the table is being stored (for building the archive
179   *                      path)
180   * @param regionDirList {@link Path} to where regions are being stored (for building the archive
181   *                      path)
182   * @throws IOException if the request cannot be completed
183   */
184  public static void archiveRegions(Configuration conf, FileSystem fs, Path rootDir, Path tableDir,
185    List<Path> regionDirList) throws IOException {
186    List<Future<Void>> futures = new ArrayList<>(regionDirList.size());
187    for (Path regionDir : regionDirList) {
188      Future<Void> future = getArchiveExecutor(conf).submit(() -> {
189        archiveRegion(fs, rootDir, tableDir, regionDir);
190        return null;
191      });
192      futures.add(future);
193    }
194    try {
195      for (Future<Void> future : futures) {
196        future.get();
197      }
198    } catch (InterruptedException e) {
199      throw new InterruptedIOException(e.getMessage());
200    } catch (ExecutionException e) {
201      throw new IOException(e.getCause());
202    }
203  }
204
205  private static synchronized ThreadPoolExecutor getArchiveExecutor(final Configuration conf) {
206    if (archiveExecutor == null) {
207      int maxThreads = conf.getInt("hbase.hfilearchiver.thread.pool.max", 8);
208      archiveExecutor =
209        Threads.getBoundedCachedThreadPool(maxThreads, 30L, TimeUnit.SECONDS, getThreadFactory());
210
211      // Shutdown this ThreadPool in a shutdown hook
212      Runtime.getRuntime().addShutdownHook(new Thread(() -> archiveExecutor.shutdown()));
213    }
214    return archiveExecutor;
215  }
216
217  // We need this method instead of Threads.getNamedThreadFactory() to pass some tests.
218  // The difference from Threads.getNamedThreadFactory() is that it doesn't fix ThreadGroup for
219  // new threads. If we use Threads.getNamedThreadFactory(), we will face ThreadGroup related
220  // issues in some tests.
221  private static ThreadFactory getThreadFactory() {
222    return new ThreadFactory() {
223      final AtomicInteger threadNumber = new AtomicInteger(1);
224
225      @Override
226      public Thread newThread(Runnable r) {
227        final String name = "HFileArchiver-" + threadNumber.getAndIncrement();
228        Thread t = new Thread(r, name);
229        t.setDaemon(true);
230        return t;
231      }
232    };
233  }
234
235  /**
236   * Remove from the specified region the store files of the specified column family, either by
237   * archiving them or outright deletion
238   * @param fs       the filesystem where the store files live
239   * @param conf     {@link Configuration} to examine to determine the archive directory
240   * @param parent   Parent region hosting the store files
241   * @param tableDir {@link Path} to where the table is being stored (for building the archive path)
242   * @param family   the family hosting the store files
243   * @throws IOException if the files could not be correctly disposed.
244   */
245  public static void archiveFamily(FileSystem fs, Configuration conf, RegionInfo parent,
246    Path tableDir, byte[] family) throws IOException {
247    Path familyDir = new Path(tableDir, new Path(parent.getEncodedName(), Bytes.toString(family)));
248    archiveFamilyByFamilyDir(fs, conf, parent, familyDir, family);
249  }
250
251  /**
252   * Removes from the specified region the store files of the specified column family, either by
253   * archiving them or outright deletion
254   * @param fs        the filesystem where the store files live
255   * @param conf      {@link Configuration} to examine to determine the archive directory
256   * @param parent    Parent region hosting the store files
257   * @param familyDir {@link Path} to where the family is being stored
258   * @param family    the family hosting the store files
259   * @throws IOException if the files could not be correctly disposed.
260   */
261  public static void archiveFamilyByFamilyDir(FileSystem fs, Configuration conf, RegionInfo parent,
262    Path familyDir, byte[] family) throws IOException {
263    FileStatus[] storeFiles = CommonFSUtils.listStatus(fs, familyDir);
264    if (storeFiles == null) {
265      LOG.debug("No files to dispose of in {}, family={}", parent.getRegionNameAsString(),
266        Bytes.toString(family));
267      return;
268    }
269
270    FileStatusConverter getAsFile = new FileStatusConverter(fs);
271    Collection<File> toArchive = Stream.of(storeFiles).map(getAsFile).collect(Collectors.toList());
272    Path storeArchiveDir = HFileArchiveUtil.getStoreArchivePath(conf, parent, family);
273
274    // do the actual archive
275    List<File> failedArchive =
276      resolveAndArchive(fs, storeArchiveDir, toArchive, EnvironmentEdgeManager.currentTime());
277    if (!failedArchive.isEmpty()) {
278      throw new FailedArchiveException(
279        "Failed to archive/delete all the files for region:"
280          + Bytes.toString(parent.getRegionName()) + ", family:" + Bytes.toString(family) + " into "
281          + storeArchiveDir + ". Something is probably awry on the filesystem.",
282        failedArchive.stream().map(FUNC_FILE_TO_PATH).collect(Collectors.toList()));
283    }
284  }
285
286  /**
287   * Remove the store files, either by archiving them or outright deletion
288   * @param conf           {@link Configuration} to examine to determine the archive directory
289   * @param fs             the filesystem where the store files live
290   * @param regionInfo     {@link RegionInfo} of the region hosting the store files
291   * @param family         the family hosting the store files
292   * @param compactedFiles files to be disposed of. No further reading of these files should be
293   *                       attempted; otherwise likely to cause an {@link IOException}
294   * @throws IOException if the files could not be correctly disposed.
295   */
296  public static void archiveStoreFiles(Configuration conf, FileSystem fs, RegionInfo regionInfo,
297    Path tableDir, byte[] family, Collection<HStoreFile> compactedFiles) throws IOException {
298    Path storeArchiveDir = HFileArchiveUtil.getStoreArchivePath(conf, regionInfo, tableDir, family);
299    archive(fs, regionInfo, family, compactedFiles, storeArchiveDir);
300  }
301
302  /**
303   * Archive recovered edits using existing logic for archiving store files. This is currently only
304   * relevant when <b>hbase.region.archive.recovered.edits</b> is true, as recovered edits shouldn't
305   * be kept after replay. In theory, we could use very same method available for archiving store
306   * files, but supporting WAL dir and store files on different FileSystems added the need for extra
307   * validation of the passed FileSystem instance and the path where the archiving edits should be
308   * placed.
309   * @param conf          {@link Configuration} to determine the archive directory.
310   * @param fs            the filesystem used for storing WAL files.
311   * @param regionInfo    {@link RegionInfo} a pseudo region representation for the archiving logic.
312   * @param family        a pseudo familiy representation for the archiving logic.
313   * @param replayedEdits the recovered edits to be archived.
314   * @throws IOException if files can't be achived due to some internal error.
315   */
316  public static void archiveRecoveredEdits(Configuration conf, FileSystem fs, RegionInfo regionInfo,
317    byte[] family, Collection<HStoreFile> replayedEdits) throws IOException {
318    String workingDir = conf.get(CommonFSUtils.HBASE_WAL_DIR, conf.get(HConstants.HBASE_DIR));
319    // extra sanity checks for the right FS
320    Path path = new Path(workingDir);
321    if (path.isAbsoluteAndSchemeAuthorityNull()) {
322      // no schema specified on wal dir value, so it's on same FS as StoreFiles
323      path = new Path(conf.get(HConstants.HBASE_DIR));
324    }
325    if (path.toUri().getScheme() != null && !path.toUri().getScheme().equals(fs.getScheme())) {
326      throw new IOException(
327        "Wrong file system! Should be " + path.toUri().getScheme() + ", but got " + fs.getScheme());
328    }
329    path = HFileArchiveUtil.getStoreArchivePathForRootDir(path, regionInfo, family);
330    archive(fs, regionInfo, family, replayedEdits, path);
331  }
332
333  private static void archive(FileSystem fs, RegionInfo regionInfo, byte[] family,
334    Collection<HStoreFile> compactedFiles, Path storeArchiveDir) throws IOException {
335    // sometimes in testing, we don't have rss, so we need to check for that
336    if (fs == null) {
337      LOG.warn(
338        "Passed filesystem is null, so just deleting files without archiving for {}," + "family={}",
339        Bytes.toString(regionInfo.getRegionName()), Bytes.toString(family));
340      deleteStoreFilesWithoutArchiving(compactedFiles);
341      return;
342    }
343
344    // short circuit if we don't have any files to delete
345    if (compactedFiles.isEmpty()) {
346      LOG.debug("No files to dispose of, done!");
347      return;
348    }
349
350    // build the archive path
351    if (regionInfo == null || family == null)
352      throw new IOException("Need to have a region and a family to archive from.");
353    // make sure we don't archive if we can't and that the archive dir exists
354    if (!fs.mkdirs(storeArchiveDir)) {
355      throw new IOException("Could not make archive directory (" + storeArchiveDir + ") for store:"
356        + Bytes.toString(family) + ", deleting compacted files instead.");
357    }
358
359    // otherwise we attempt to archive the store files
360    LOG.debug("Archiving compacted files.");
361
362    // Wrap the storefile into a File
363    StoreToFile getStorePath = new StoreToFile(fs);
364    Collection<File> storeFiles =
365      compactedFiles.stream().map(getStorePath).collect(Collectors.toList());
366
367    // do the actual archive
368    List<File> failedArchive =
369      resolveAndArchive(fs, storeArchiveDir, storeFiles, EnvironmentEdgeManager.currentTime());
370
371    if (!failedArchive.isEmpty()) {
372      throw new FailedArchiveException(
373        "Failed to archive/delete all the files for region:"
374          + Bytes.toString(regionInfo.getRegionName()) + ", family:" + Bytes.toString(family)
375          + " into " + storeArchiveDir + ". Something is probably awry on the filesystem.",
376        failedArchive.stream().map(FUNC_FILE_TO_PATH).collect(Collectors.toList()));
377    }
378  }
379
380  /**
381   * Archive the store file
382   * @param fs         the filesystem where the store files live
383   * @param regionInfo region hosting the store files
384   * @param conf       {@link Configuration} to examine to determine the archive directory
385   * @param tableDir   {@link Path} to where the table is being stored (for building the archive
386   *                   path)
387   * @param family     the family hosting the store files
388   * @param storeFile  file to be archived
389   * @throws IOException if the files could not be correctly disposed.
390   */
391  public static void archiveStoreFile(Configuration conf, FileSystem fs, RegionInfo regionInfo,
392    Path tableDir, byte[] family, Path storeFile) throws IOException {
393    Path storeArchiveDir = HFileArchiveUtil.getStoreArchivePath(conf, regionInfo, tableDir, family);
394    // make sure we don't archive if we can't and that the archive dir exists
395    if (!fs.mkdirs(storeArchiveDir)) {
396      throw new IOException("Could not make archive directory (" + storeArchiveDir + ") for store:"
397        + Bytes.toString(family) + ", deleting compacted files instead.");
398    }
399
400    // do the actual archive
401    long start = EnvironmentEdgeManager.currentTime();
402    File file = new FileablePath(fs, storeFile);
403    if (!resolveAndArchiveFile(storeArchiveDir, file, Long.toString(start))) {
404      throw new IOException("Failed to archive/delete the file for region:"
405        + regionInfo.getRegionNameAsString() + ", family:" + Bytes.toString(family) + " into "
406        + storeArchiveDir + ". Something is probably awry on the filesystem.");
407    }
408  }
409
410  /**
411   * Resolve any conflict with an existing archive file via timestamp-append renaming of the
412   * existing file and then archive the passed in files.
413   * @param fs             {@link FileSystem} on which to archive the files
414   * @param baseArchiveDir base archive directory to store the files. If any of the files to archive
415   *                       are directories, will append the name of the directory to the base
416   *                       archive directory name, creating a parallel structure.
417   * @param toArchive      files/directories that need to be archvied
418   * @param start          time the archiving started - used for resolving archive conflicts.
419   * @return the list of failed to archive files.
420   * @throws IOException if an unexpected file operation exception occurred
421   */
422  private static List<File> resolveAndArchive(FileSystem fs, Path baseArchiveDir,
423    Collection<File> toArchive, long start) throws IOException {
424    // short circuit if no files to move
425    if (toArchive.isEmpty()) {
426      return Collections.emptyList();
427    }
428
429    LOG.trace("Moving files to the archive directory {}", baseArchiveDir);
430
431    // make sure the archive directory exists
432    if (!fs.exists(baseArchiveDir)) {
433      if (!fs.mkdirs(baseArchiveDir)) {
434        throw new IOException("Failed to create the archive directory:" + baseArchiveDir
435          + ", quitting archive attempt.");
436      }
437      LOG.trace("Created archive directory {}", baseArchiveDir);
438    }
439
440    List<File> failures = new ArrayList<>();
441    String startTime = Long.toString(start);
442    for (File file : toArchive) {
443      // if its a file archive it
444      try {
445        LOG.trace("Archiving {}", file);
446        if (file.isFile()) {
447          // attempt to archive the file
448          if (!resolveAndArchiveFile(baseArchiveDir, file, startTime)) {
449            LOG.warn("Couldn't archive " + file + " into backup directory: " + baseArchiveDir);
450            failures.add(file);
451          }
452        } else {
453          // otherwise its a directory and we need to archive all files
454          LOG.trace("{} is a directory, archiving children files", file);
455          // so we add the directory name to the one base archive
456          Path parentArchiveDir = new Path(baseArchiveDir, file.getName());
457          // and then get all the files from that directory and attempt to
458          // archive those too
459          Collection<File> children = file.getChildren();
460          failures.addAll(resolveAndArchive(fs, parentArchiveDir, children, start));
461        }
462      } catch (IOException e) {
463        LOG.warn("Failed to archive {}", file, e);
464        failures.add(file);
465      }
466    }
467    return failures;
468  }
469
470  /**
471   * Attempt to archive the passed in file to the archive directory.
472   * <p>
473   * If the same file already exists in the archive, it is moved to a timestamped directory under
474   * the archive directory and the new file is put in its place.
475   * @param archiveDir       {@link Path} to the directory that stores the archives of the hfiles
476   * @param currentFile      {@link Path} to the original HFile that will be archived
477   * @param archiveStartTime time the archiving started, to resolve naming conflicts
478   * @return <tt>true</tt> if the file is successfully archived. <tt>false</tt> if there was a
479   *         problem, but the operation still completed.
480   * @throws IOException on failure to complete {@link FileSystem} operations.
481   */
482  private static boolean resolveAndArchiveFile(Path archiveDir, File currentFile,
483    String archiveStartTime) throws IOException {
484    // build path as it should be in the archive
485    String filename = currentFile.getName();
486    Path archiveFile = new Path(archiveDir, filename);
487    FileSystem fs = currentFile.getFileSystem();
488
489    // An existing destination file in the archive is unexpected, but we handle it here.
490    if (fs.exists(archiveFile)) {
491      if (!fs.exists(currentFile.getPath())) {
492        // If the file already exists in the archive, and there is no current file to archive, then
493        // assume that the file in archive is correct. This is an unexpected situation, suggesting a
494        // race condition or split brain.
495        // In HBASE-26718 this was found when compaction incorrectly happened during warmupRegion.
496        LOG.warn("{} exists in archive. Attempted to archive nonexistent file {}.", archiveFile,
497          currentFile);
498        // We return success to match existing behavior in this method, where FileNotFoundException
499        // in moveAndClose is ignored.
500        return true;
501      }
502      // There is a conflict between the current file and the already existing archived file.
503      // Move the archived file to a timestamped backup. This is a really, really unlikely
504      // situation, where we get the same name for the existing file, but is included just for that
505      // 1 in trillion chance. We are potentially incurring data loss in the archive directory if
506      // the files are not identical. The timestamped backup will be cleaned by HFileCleaner as it
507      // has no references.
508      FileStatus curStatus = fs.getFileStatus(currentFile.getPath());
509      FileStatus archiveStatus = fs.getFileStatus(archiveFile);
510      long curLen = curStatus.getLen();
511      long archiveLen = archiveStatus.getLen();
512      long curMtime = curStatus.getModificationTime();
513      long archiveMtime = archiveStatus.getModificationTime();
514      if (curLen != archiveLen) {
515        LOG.error(
516          "{} already exists in archive with different size than current {}."
517            + " archiveLen: {} currentLen: {} archiveMtime: {} currentMtime: {}",
518          archiveFile, currentFile, archiveLen, curLen, archiveMtime, curMtime);
519        throw new IOException(
520          archiveFile + " already exists in archive with different size" + " than " + currentFile);
521      }
522
523      LOG.error(
524        "{} already exists in archive, moving to timestamped backup and overwriting"
525          + " current {}. archiveLen: {} currentLen: {} archiveMtime: {} currentMtime: {}",
526        archiveFile, currentFile, archiveLen, curLen, archiveMtime, curMtime);
527
528      // move the archive file to the stamped backup
529      Path backedupArchiveFile = new Path(archiveDir, filename + SEPARATOR + archiveStartTime);
530      if (!fs.rename(archiveFile, backedupArchiveFile)) {
531        LOG.error("Could not rename archive file to backup: " + backedupArchiveFile
532          + ", deleting existing file in favor of newer.");
533        // try to delete the existing file, if we can't rename it
534        if (!fs.delete(archiveFile, false)) {
535          throw new IOException("Couldn't delete existing archive file (" + archiveFile
536            + ") or rename it to the backup file (" + backedupArchiveFile
537            + ") to make room for similarly named file.");
538        }
539      } else {
540        LOG.info("Backed up archive file from {} to {}.", archiveFile, backedupArchiveFile);
541      }
542    }
543
544    LOG.trace("No existing file in archive for {}, free to archive original file.", archiveFile);
545
546    // at this point, we should have a free spot for the archive file
547    boolean success = false;
548    for (int i = 0; !success && i < DEFAULT_RETRIES_NUMBER; ++i) {
549      if (i > 0) {
550        // Ensure that the archive directory exists.
551        // The previous "move to archive" operation has failed probably because
552        // the cleaner has removed our archive directory (HBASE-7643).
553        // (we're in a retry loop, so don't worry too much about the exception)
554        try {
555          if (!fs.exists(archiveDir)) {
556            if (fs.mkdirs(archiveDir)) {
557              LOG.debug("Created archive directory {}", archiveDir);
558            }
559          }
560        } catch (IOException e) {
561          LOG.warn("Failed to create directory {}", archiveDir, e);
562        }
563      }
564
565      try {
566        success = currentFile.moveAndClose(archiveFile);
567      } catch (FileNotFoundException fnfe) {
568        LOG.warn("Failed to archive " + currentFile
569          + " because it does not exist! Skipping and continuing on.", fnfe);
570        success = true;
571      } catch (IOException e) {
572        success = false;
573        // When HFiles are placed on a filesystem other than HDFS a rename operation can be a
574        // non-atomic file copy operation. It can take a long time to copy a large hfile and if
575        // interrupted there may be a partially copied file present at the destination. We must
576        // remove the partially copied file, if any, or otherwise the archive operation will fail
577        // indefinitely from this point.
578        LOG.warn("Failed to archive " + currentFile + " on try #" + i, e);
579        try {
580          fs.delete(archiveFile, false);
581        } catch (FileNotFoundException fnfe) {
582          // This case is fine.
583        } catch (IOException ee) {
584          // Complain about other IO exceptions
585          LOG.warn("Failed to clean up from failure to archive " + currentFile + " on try #" + i,
586            ee);
587        }
588      }
589    }
590
591    if (!success) {
592      LOG.error("Failed to archive " + currentFile);
593      return false;
594    }
595
596    LOG.debug("Archived from {} to {}", currentFile, archiveFile);
597    return true;
598  }
599
600  /**
601   * Without regard for backup, delete a region. Should be used with caution.
602   * @param regionDir {@link Path} to the region to be deleted.
603   * @param fs        FileSystem from which to delete the region
604   * @return <tt>true</tt> on successful deletion, <tt>false</tt> otherwise
605   * @throws IOException on filesystem operation failure
606   */
607  private static boolean deleteRegionWithoutArchiving(FileSystem fs, Path regionDir)
608    throws IOException {
609    if (fs.delete(regionDir, true)) {
610      LOG.debug("Deleted {}", regionDir);
611      return true;
612    }
613    LOG.debug("Failed to delete directory {}", regionDir);
614    return false;
615  }
616
617  /**
618   * Just do a simple delete of the given store files
619   * <p>
620   * A best effort is made to delete each of the files, rather than bailing on the first failure.
621   * <p>
622   * @param compactedFiles store files to delete from the file system.
623   * @throws IOException if a file cannot be deleted. All files will be attempted to deleted before
624   *                     throwing the exception, rather than failing at the first file.
625   */
626  private static void deleteStoreFilesWithoutArchiving(Collection<HStoreFile> compactedFiles)
627    throws IOException {
628    LOG.debug("Deleting files without archiving.");
629    List<IOException> errors = new ArrayList<>(0);
630    for (HStoreFile hsf : compactedFiles) {
631      try {
632        hsf.deleteStoreFile();
633      } catch (IOException e) {
634        LOG.error("Failed to delete {}", hsf.getPath());
635        errors.add(e);
636      }
637    }
638    if (errors.size() > 0) {
639      throw MultipleIOException.createIOException(errors);
640    }
641  }
642
643  /**
644   * Adapt a type to match the {@link File} interface, which is used internally for handling
645   * archival/removal of files
646   * @param <T> type to adapt to the {@link File} interface
647   */
648  private static abstract class FileConverter<T> implements Function<T, File> {
649    protected final FileSystem fs;
650
651    public FileConverter(FileSystem fs) {
652      this.fs = fs;
653    }
654  }
655
656  /**
657   * Convert a FileStatus to something we can manage in the archiving
658   */
659  private static class FileStatusConverter extends FileConverter<FileStatus> {
660    public FileStatusConverter(FileSystem fs) {
661      super(fs);
662    }
663
664    @Override
665    public File apply(FileStatus input) {
666      return new FileablePath(fs, input.getPath());
667    }
668  }
669
670  /**
671   * Convert the {@link HStoreFile} into something we can manage in the archive methods
672   */
673  private static class StoreToFile extends FileConverter<HStoreFile> {
674    public StoreToFile(FileSystem fs) {
675      super(fs);
676    }
677
678    @Override
679    public File apply(HStoreFile input) {
680      return new FileableStoreFile(fs, input);
681    }
682  }
683
684  /**
685   * Wrapper to handle file operations uniformly
686   */
687  private static abstract class File {
688    protected final FileSystem fs;
689
690    public File(FileSystem fs) {
691      this.fs = fs;
692    }
693
694    /**
695     * Delete the file
696     * @throws IOException on failure
697     */
698    abstract void delete() throws IOException;
699
700    /**
701     * Check to see if this is a file or a directory
702     * @return <tt>true</tt> if it is a file, <tt>false</tt> otherwise
703     * @throws IOException on {@link FileSystem} connection error
704     */
705    abstract boolean isFile() throws IOException;
706
707    /**
708     * @return if this is a directory, returns all the children in the directory, otherwise returns
709     *         an empty list
710     */
711    abstract Collection<File> getChildren() throws IOException;
712
713    /**
714     * close any outside readers of the file
715     */
716    abstract void close() throws IOException;
717
718    /** Returns the name of the file (not the full fs path, just the individual file name) */
719    abstract String getName();
720
721    /** Returns the path to this file */
722    abstract Path getPath();
723
724    /**
725     * Move the file to the given destination
726     * @return <tt>true</tt> on success
727     */
728    public boolean moveAndClose(Path dest) throws IOException {
729      this.close();
730      Path p = this.getPath();
731      return CommonFSUtils.renameAndSetModifyTime(fs, p, dest);
732    }
733
734    /** Returns the {@link FileSystem} on which this file resides */
735    public FileSystem getFileSystem() {
736      return this.fs;
737    }
738
739    @Override
740    public String toString() {
741      return this.getClass().getSimpleName() + ", " + getPath().toString();
742    }
743  }
744
745  /**
746   * A {@link File} that wraps a simple {@link Path} on a {@link FileSystem}.
747   */
748  private static class FileablePath extends File {
749    private final Path file;
750    private final FileStatusConverter getAsFile;
751
752    public FileablePath(FileSystem fs, Path file) {
753      super(fs);
754      this.file = file;
755      this.getAsFile = new FileStatusConverter(fs);
756    }
757
758    @Override
759    public void delete() throws IOException {
760      if (!fs.delete(file, true)) throw new IOException("Failed to delete:" + this.file);
761    }
762
763    @Override
764    public String getName() {
765      return file.getName();
766    }
767
768    @Override
769    public Collection<File> getChildren() throws IOException {
770      if (fs.isFile(file)) {
771        return Collections.emptyList();
772      }
773      return Stream.of(fs.listStatus(file)).map(getAsFile).collect(Collectors.toList());
774    }
775
776    @Override
777    public boolean isFile() throws IOException {
778      return fs.isFile(file);
779    }
780
781    @Override
782    public void close() throws IOException {
783      // NOOP - files are implicitly closed on removal
784    }
785
786    @Override
787    Path getPath() {
788      return file;
789    }
790  }
791
792  /**
793   * {@link File} adapter for a {@link HStoreFile} living on a {@link FileSystem} .
794   */
795  private static class FileableStoreFile extends File {
796    HStoreFile file;
797
798    public FileableStoreFile(FileSystem fs, HStoreFile store) {
799      super(fs);
800      this.file = store;
801    }
802
803    @Override
804    public void delete() throws IOException {
805      file.deleteStoreFile();
806    }
807
808    @Override
809    public String getName() {
810      return file.getPath().getName();
811    }
812
813    @Override
814    public boolean isFile() {
815      return true;
816    }
817
818    @Override
819    public Collection<File> getChildren() throws IOException {
820      // storefiles don't have children
821      return Collections.emptyList();
822    }
823
824    @Override
825    public void close() throws IOException {
826      file.closeStoreFile(true);
827    }
828
829    @Override
830    Path getPath() {
831      return file.getPath();
832    }
833  }
834}