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.snapshot;
019
020import java.io.FileNotFoundException;
021import java.io.IOException;
022import java.net.URI;
023import java.text.SimpleDateFormat;
024import java.util.ArrayList;
025import java.util.Collections;
026import java.util.Date;
027import java.util.List;
028import java.util.Map;
029import java.util.Properties;
030import java.util.concurrent.ConcurrentHashMap;
031import java.util.concurrent.ExecutorService;
032import java.util.concurrent.atomic.AtomicInteger;
033import java.util.concurrent.atomic.AtomicLong;
034import org.apache.hadoop.conf.Configuration;
035import org.apache.hadoop.fs.FileStatus;
036import org.apache.hadoop.fs.FileSystem;
037import org.apache.hadoop.fs.Path;
038import org.apache.hadoop.hbase.TableName;
039import org.apache.hadoop.hbase.client.RegionInfo;
040import org.apache.hadoop.hbase.client.SnapshotDescription;
041import org.apache.hadoop.hbase.io.HFileLink;
042import org.apache.hadoop.hbase.io.WALLink;
043import org.apache.hadoop.hbase.util.AbstractHBaseTool;
044import org.apache.hadoop.hbase.util.CommonFSUtils;
045import org.apache.hadoop.util.StringUtils;
046import org.apache.yetus.audience.InterfaceAudience;
047import org.slf4j.Logger;
048import org.slf4j.LoggerFactory;
049
050import org.apache.hbase.thirdparty.org.apache.commons.cli.CommandLine;
051import org.apache.hbase.thirdparty.org.apache.commons.cli.CommandLineParser;
052import org.apache.hbase.thirdparty.org.apache.commons.cli.DefaultParser;
053import org.apache.hbase.thirdparty.org.apache.commons.cli.Option;
054import org.apache.hbase.thirdparty.org.apache.commons.cli.Options;
055import org.apache.hbase.thirdparty.org.apache.commons.cli.ParseException;
056
057import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
058import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos;
059import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotRegionManifest;
060
061/**
062 * Tool for dumping snapshot information.
063 * <ol>
064 * <li>Table Descriptor
065 * <li>Snapshot creation time, type, format version, ...
066 * <li>List of hfiles and wals
067 * <li>Stats about hfiles and logs sizes, percentage of shared with the source table, ...
068 * </ol>
069 */
070@InterfaceAudience.Public
071public final class SnapshotInfo extends AbstractHBaseTool {
072  private static final Logger LOG = LoggerFactory.getLogger(SnapshotInfo.class);
073
074  static final class Options {
075    static final Option SNAPSHOT =
076      new Option(null, "snapshot", true, "The name of the snapshot to be detailed.");
077    static final Option REMOTE_DIR =
078      new Option(null, "remote-dir", true, "A custom root directory where snapshots are stored. "
079        + "Use it together with the --snapshot option.");
080    static final Option LIST_SNAPSHOTS =
081      new Option(null, "list-snapshots", false, "List all the available snapshots and exit.");
082    static final Option FILES =
083      new Option(null, "files", false, "The list of files retained by the specified snapshot. "
084        + "Use it together with the --snapshot option.");
085    static final Option STATS =
086      new Option(null, "stats", false, "Additional information about the specified snapshot. "
087        + "Use it together with the --snapshot option.");
088    static final Option SCHEMA = new Option(null, "schema", false,
089      "Show the descriptor of the table for the specified snapshot. "
090        + "Use it together with the --snapshot option.");
091    static final Option SIZE_IN_BYTES =
092      new Option(null, "size-in-bytes", false, "Print the size of the files in bytes. "
093        + "Use it together with the --snapshot and --files options.");
094  }
095
096  /**
097   * Statistics about the snapshot
098   * <ol>
099   * <li>How many store files and logs are in the archive
100   * <li>How many store files and logs are shared with the table
101   * <li>Total store files and logs size and shared amount
102   * </ol>
103   */
104  public static class SnapshotStats {
105    /** Information about the file referenced by the snapshot */
106    static class FileInfo {
107      private final boolean corrupted;
108      private final boolean inArchive;
109      private final long size;
110
111      FileInfo(final boolean inArchive, final long size, final boolean corrupted) {
112        this.corrupted = corrupted;
113        this.inArchive = inArchive;
114        this.size = size;
115      }
116
117      /** Returns true if the file is in the archive */
118      public boolean inArchive() {
119        return this.inArchive;
120      }
121
122      /** Returns true if the file is corrupted */
123      public boolean isCorrupted() {
124        return this.corrupted;
125      }
126
127      /** Returns true if the file is missing */
128      public boolean isMissing() {
129        return this.size < 0;
130      }
131
132      /** Returns the file size */
133      public long getSize() {
134        return this.size;
135      }
136
137      String getStateToString() {
138        if (isCorrupted()) return "CORRUPTED";
139        if (isMissing()) return "NOT FOUND";
140        if (inArchive()) return "archive";
141        return null;
142      }
143    }
144
145    private AtomicInteger hfilesArchiveCount = new AtomicInteger();
146    private AtomicInteger hfilesCorrupted = new AtomicInteger();
147    private AtomicInteger hfilesMissing = new AtomicInteger();
148    private AtomicInteger hfilesCount = new AtomicInteger();
149    private AtomicInteger hfilesMobCount = new AtomicInteger();
150    private AtomicInteger logsMissing = new AtomicInteger();
151    private AtomicInteger logsCount = new AtomicInteger();
152    private AtomicLong hfilesArchiveSize = new AtomicLong();
153    private AtomicLong hfilesSize = new AtomicLong();
154    private AtomicLong hfilesMobSize = new AtomicLong();
155    private AtomicLong nonSharedHfilesArchiveSize = new AtomicLong();
156    private AtomicLong logSize = new AtomicLong();
157
158    private final SnapshotProtos.SnapshotDescription snapshot;
159    private final TableName snapshotTable;
160    private final Configuration conf;
161    private final FileSystem fs;
162
163    SnapshotStats(final Configuration conf, final FileSystem fs,
164      final SnapshotDescription snapshot) {
165      this.snapshot = ProtobufUtil.createHBaseProtosSnapshotDesc(snapshot);
166      this.snapshotTable = snapshot.getTableName();
167      this.conf = conf;
168      this.fs = fs;
169    }
170
171    SnapshotStats(final Configuration conf, final FileSystem fs,
172      final SnapshotProtos.SnapshotDescription snapshot) {
173      this.snapshot = snapshot;
174      this.snapshotTable = TableName.valueOf(snapshot.getTable());
175      this.conf = conf;
176      this.fs = fs;
177    }
178
179    /** Returns the snapshot descriptor */
180    public SnapshotDescription getSnapshotDescription() {
181      return ProtobufUtil.createSnapshotDesc(this.snapshot);
182    }
183
184    /** Returns true if the snapshot is corrupted */
185    public boolean isSnapshotCorrupted() {
186      return hfilesMissing.get() > 0 || logsMissing.get() > 0 || hfilesCorrupted.get() > 0;
187    }
188
189    /** Returns the number of available store files */
190    public int getStoreFilesCount() {
191      return hfilesCount.get() + hfilesArchiveCount.get() + hfilesMobCount.get();
192    }
193
194    /** Returns the number of available store files in the archive */
195    public int getArchivedStoreFilesCount() {
196      return hfilesArchiveCount.get();
197    }
198
199    /** Returns the number of available store files in the mob dir */
200    public int getMobStoreFilesCount() {
201      return hfilesMobCount.get();
202    }
203
204    /** Returns the number of available log files */
205    public int getLogsCount() {
206      return logsCount.get();
207    }
208
209    /** Returns the number of missing store files */
210    public int getMissingStoreFilesCount() {
211      return hfilesMissing.get();
212    }
213
214    /** Returns the number of corrupted store files */
215    public int getCorruptedStoreFilesCount() {
216      return hfilesCorrupted.get();
217    }
218
219    /** Returns the number of missing log files */
220    public int getMissingLogsCount() {
221      return logsMissing.get();
222    }
223
224    /** Returns the total size of the store files referenced by the snapshot */
225    public long getStoreFilesSize() {
226      return hfilesSize.get() + hfilesArchiveSize.get() + hfilesMobSize.get();
227    }
228
229    /** Returns the total size of the store files shared */
230    public long getSharedStoreFilesSize() {
231      return hfilesSize.get();
232    }
233
234    /** Returns the total size of the store files in the archive */
235    public long getArchivedStoreFileSize() {
236      return hfilesArchiveSize.get();
237    }
238
239    /** Returns the total size of the store files in the mob store */
240    public long getMobStoreFilesSize() {
241      return hfilesMobSize.get();
242    }
243
244    /**
245     * @return the total size of the store files in the archive which is not shared with other
246     *         snapshots and tables This is only calculated when
247     *         {@link #getSnapshotStats(Configuration, SnapshotProtos.SnapshotDescription, Map)} is
248     *         called with a non-null Map
249     */
250    public long getNonSharedArchivedStoreFilesSize() {
251      return nonSharedHfilesArchiveSize.get();
252    }
253
254    /** Returns the percentage of the shared store files */
255    public float getSharedStoreFilePercentage() {
256      return ((float) hfilesSize.get() / (getStoreFilesSize())) * 100;
257    }
258
259    /** Returns the percentage of the mob store files */
260    public float getMobStoreFilePercentage() {
261      return ((float) hfilesMobSize.get() / (getStoreFilesSize())) * 100;
262    }
263
264    /** Returns the total log size */
265    public long getLogsSize() {
266      return logSize.get();
267    }
268
269    /**
270     * Check if for a give file in archive, if there are other snapshots/tables still reference it.
271     * @param filePath         file path in archive
272     * @param snapshotFilesMap a map for store files in snapshots about how many snapshots refer to
273     *                         it.
274     * @return true or false
275     */
276    private boolean isArchivedFileStillReferenced(final Path filePath,
277      final Map<Path, Integer> snapshotFilesMap) {
278
279      Integer c = snapshotFilesMap.get(filePath);
280
281      // Check if there are other snapshots or table from clone_snapshot() (via back-reference)
282      // still reference to it.
283      if ((c != null) && (c == 1)) {
284        Path parentDir = filePath.getParent();
285        Path backRefDir = HFileLink.getBackReferencesDir(parentDir, filePath.getName());
286        try {
287          if (CommonFSUtils.listStatus(fs, backRefDir) == null) {
288            return false;
289          }
290        } catch (IOException e) {
291          // For the purpose of this function, IOException is ignored and treated as
292          // the file is still being referenced.
293        }
294      }
295      return true;
296    }
297
298    /**
299     * Add the specified store file to the stats
300     * @param region    region encoded Name
301     * @param family    family name
302     * @param storeFile store file name
303     * @param filesMap  store files map for all snapshots, it may be null
304     * @return the store file information
305     */
306    FileInfo addStoreFile(final RegionInfo region, final String family,
307      final SnapshotRegionManifest.StoreFile storeFile, final Map<Path, Integer> filesMap)
308      throws IOException {
309      HFileLink link =
310        HFileLink.build(conf, snapshotTable, region.getEncodedName(), family, storeFile.getName());
311      boolean isCorrupted = false;
312      boolean inArchive = false;
313      long size = -1;
314      try {
315        if (fs.exists(link.getArchivePath())) {
316          inArchive = true;
317          size = fs.getFileStatus(link.getArchivePath()).getLen();
318          hfilesArchiveSize.addAndGet(size);
319          hfilesArchiveCount.incrementAndGet();
320
321          // If store file is not shared with other snapshots and tables,
322          // increase nonSharedHfilesArchiveSize
323          if (
324            (filesMap != null) && !isArchivedFileStillReferenced(link.getArchivePath(), filesMap)
325          ) {
326            nonSharedHfilesArchiveSize.addAndGet(size);
327          }
328        } else if (fs.exists(link.getMobPath())) {
329          inArchive = true;
330          size = fs.getFileStatus(link.getMobPath()).getLen();
331          hfilesMobSize.addAndGet(size);
332          hfilesMobCount.incrementAndGet();
333        } else {
334          size = link.getFileStatus(fs).getLen();
335          hfilesSize.addAndGet(size);
336          hfilesCount.incrementAndGet();
337        }
338        isCorrupted = (storeFile.hasFileSize() && storeFile.getFileSize() != size);
339        if (isCorrupted) hfilesCorrupted.incrementAndGet();
340      } catch (FileNotFoundException e) {
341        hfilesMissing.incrementAndGet();
342      }
343      return new FileInfo(inArchive, size, isCorrupted);
344    }
345
346    /**
347     * Add the specified log file to the stats
348     * @param server  server name
349     * @param logfile log file name
350     * @return the log information
351     */
352    FileInfo addLogFile(final String server, final String logfile) throws IOException {
353      WALLink logLink = new WALLink(conf, server, logfile);
354      long size = -1;
355      try {
356        size = logLink.getFileStatus(fs).getLen();
357        logSize.addAndGet(size);
358        logsCount.incrementAndGet();
359      } catch (FileNotFoundException e) {
360        logsMissing.incrementAndGet();
361      }
362      return new FileInfo(false, size, false);
363    }
364  }
365
366  private FileSystem fs;
367  private Path rootDir;
368
369  private SnapshotManifest snapshotManifest;
370
371  private boolean listSnapshots = false;
372  private String snapshotName;
373  private Path remoteDir;
374  private boolean showSchema = false;
375  private boolean showFiles = false;
376  private boolean showStats = false;
377  private boolean printSizeInBytes = false;
378
379  @Override
380  public int doWork() throws IOException, InterruptedException {
381    if (remoteDir != null) {
382      URI defaultFs = remoteDir.getFileSystem(conf).getUri();
383      CommonFSUtils.setFsDefault(conf, new Path(defaultFs));
384      CommonFSUtils.setRootDir(conf, remoteDir);
385    }
386
387    // List Available Snapshots
388    if (listSnapshots) {
389      SimpleDateFormat df = new SimpleDateFormat("yyyy-MM-dd'T'HH:mm:ss");
390      System.out.printf("%-20s | %-20s | %-20s | %s%n", "SNAPSHOT", "CREATION TIME", "TTL IN SEC",
391        "TABLE NAME");
392      for (SnapshotDescription desc : getSnapshotList(conf)) {
393        System.out.printf("%-20s | %20s | %20s | %s%n", desc.getName(),
394          df.format(new Date(desc.getCreationTime())), desc.getTtl(), desc.getTableNameAsString());
395      }
396      return 0;
397    }
398
399    rootDir = CommonFSUtils.getRootDir(conf);
400    fs = FileSystem.get(rootDir.toUri(), conf);
401    LOG.debug("fs=" + fs.getUri().toString() + " root=" + rootDir);
402
403    // Load snapshot information
404    if (!loadSnapshotInfo(snapshotName)) {
405      System.err.println("Snapshot '" + snapshotName + "' not found!");
406      return 1;
407    }
408
409    printInfo();
410    if (showSchema) {
411      printSchema();
412    }
413    printFiles(showFiles, showStats);
414
415    return 0;
416  }
417
418  /**
419   * Load snapshot info and table descriptor for the specified snapshot
420   * @param snapshotName name of the snapshot to load
421   * @return false if snapshot is not found
422   */
423  private boolean loadSnapshotInfo(final String snapshotName) throws IOException {
424    Path snapshotDir = SnapshotDescriptionUtils.getCompletedSnapshotDir(snapshotName, rootDir);
425    if (!fs.exists(snapshotDir)) {
426      LOG.warn("Snapshot '" + snapshotName + "' not found in: " + snapshotDir);
427      return false;
428    }
429
430    SnapshotProtos.SnapshotDescription snapshotDesc =
431      SnapshotDescriptionUtils.readSnapshotInfo(fs, snapshotDir);
432    snapshotManifest = SnapshotManifest.open(getConf(), fs, snapshotDir, snapshotDesc);
433    return true;
434  }
435
436  /**
437   * Dump the {@link SnapshotDescription}
438   */
439  private void printInfo() {
440    SnapshotProtos.SnapshotDescription snapshotDesc = snapshotManifest.getSnapshotDescription();
441    SimpleDateFormat df = new SimpleDateFormat("yyyy-MM-dd'T'HH:mm:ss");
442    System.out.println("Snapshot Info");
443    System.out.println("----------------------------------------");
444    System.out.println("   Name: " + snapshotDesc.getName());
445    System.out.println("   Type: " + snapshotDesc.getType());
446    System.out.println("  Table: " + snapshotDesc.getTable());
447    System.out.println(" Format: " + snapshotDesc.getVersion());
448    System.out.println("Created: " + df.format(new Date(snapshotDesc.getCreationTime())));
449    System.out.println("    Ttl: " + snapshotDesc.getTtl());
450    System.out.println("  Owner: " + snapshotDesc.getOwner());
451    System.out.println();
452  }
453
454  /**
455   * Dump the {@link org.apache.hadoop.hbase.client.TableDescriptor}
456   */
457  private void printSchema() {
458    System.out.println("Table Descriptor");
459    System.out.println("----------------------------------------");
460    System.out.println(snapshotManifest.getTableDescriptor().toString());
461    System.out.println();
462  }
463
464  /**
465   * Collect the hfiles and logs statistics of the snapshot and dump the file list if requested and
466   * the collected information.
467   */
468  private void printFiles(final boolean showFiles, final boolean showStats) throws IOException {
469    if (showFiles) {
470      System.out.println("Snapshot Files");
471      System.out.println("----------------------------------------");
472    }
473
474    // Collect information about hfiles and logs in the snapshot
475    final SnapshotProtos.SnapshotDescription snapshotDesc =
476      snapshotManifest.getSnapshotDescription();
477    final String table = snapshotDesc.getTable();
478    final SnapshotDescription desc = ProtobufUtil.createSnapshotDesc(snapshotDesc);
479    final SnapshotStats stats = new SnapshotStats(this.getConf(), this.fs, desc);
480    SnapshotReferenceUtil.concurrentVisitReferencedFiles(getConf(), fs, snapshotManifest,
481      "SnapshotInfo", new SnapshotReferenceUtil.SnapshotVisitor() {
482        @Override
483        public void storeFile(final RegionInfo regionInfo, final String family,
484          final SnapshotRegionManifest.StoreFile storeFile) throws IOException {
485          if (storeFile.hasReference()) return;
486
487          SnapshotStats.FileInfo info = stats.addStoreFile(regionInfo, family, storeFile, null);
488          if (showFiles) {
489            String state = info.getStateToString();
490            System.out.printf("%8s %s/%s/%s/%s %s%n",
491              (info.isMissing() ? "-" : fileSizeToString(info.getSize())), table,
492              regionInfo.getEncodedName(), family, storeFile.getName(),
493              state == null ? "" : "(" + state + ")");
494          }
495        }
496      });
497
498    // Dump the stats
499    System.out.println();
500    if (stats.isSnapshotCorrupted()) {
501      System.out.println("**************************************************************");
502      System.out.printf("BAD SNAPSHOT: %d hfile(s) and %d log(s) missing.%n",
503        stats.getMissingStoreFilesCount(), stats.getMissingLogsCount());
504      System.out.printf("              %d hfile(s) corrupted.%n",
505        stats.getCorruptedStoreFilesCount());
506      System.out.println("**************************************************************");
507    }
508
509    if (showStats) {
510      System.out.printf(
511        "%d HFiles (%d in archive, %d in mob storage), total size %s "
512          + "(%.2f%% %s shared with the source table, %.2f%% %s in mob dir)%n",
513        stats.getStoreFilesCount(), stats.getArchivedStoreFilesCount(),
514        stats.getMobStoreFilesCount(), fileSizeToString(stats.getStoreFilesSize()),
515        stats.getSharedStoreFilePercentage(), fileSizeToString(stats.getSharedStoreFilesSize()),
516        stats.getMobStoreFilePercentage(), fileSizeToString(stats.getMobStoreFilesSize()));
517      System.out.printf("%d Logs, total size %s%n", stats.getLogsCount(),
518        fileSizeToString(stats.getLogsSize()));
519      System.out.println();
520    }
521  }
522
523  private String fileSizeToString(long size) {
524    return printSizeInBytes ? Long.toString(size) : StringUtils.humanReadableInt(size);
525  }
526
527  @Override
528  protected void addOptions() {
529    addOption(Options.SNAPSHOT);
530    addOption(Options.REMOTE_DIR);
531    addOption(Options.LIST_SNAPSHOTS);
532    addOption(Options.FILES);
533    addOption(Options.STATS);
534    addOption(Options.SCHEMA);
535    addOption(Options.SIZE_IN_BYTES);
536  }
537
538  @Override
539  protected CommandLineParser newParser() {
540    // Commons-CLI lacks the capability to handle combinations of options, so we do it ourselves
541    // Validate in parse() to get helpful error messages instead of exploding in processOptions()
542    return new DefaultParser() {
543      @Override
544      public CommandLine parse(org.apache.hbase.thirdparty.org.apache.commons.cli.Options opts,
545        String[] args, Properties props, boolean stop) throws ParseException {
546        CommandLine cl = super.parse(opts, args, props, stop);
547        if (!cmd.hasOption(Options.LIST_SNAPSHOTS) && !cmd.hasOption(Options.SNAPSHOT)) {
548          throw new ParseException("Missing required snapshot option!");
549        }
550        return cl;
551      }
552    };
553  }
554
555  @Override
556  protected void processOptions(CommandLine cmd) {
557    snapshotName = cmd.getOptionValue(Options.SNAPSHOT.getLongOpt());
558    showFiles = cmd.hasOption(Options.FILES.getLongOpt());
559    showStats =
560      cmd.hasOption(Options.FILES.getLongOpt()) || cmd.hasOption(Options.STATS.getLongOpt());
561    showSchema = cmd.hasOption(Options.SCHEMA.getLongOpt());
562    listSnapshots = cmd.hasOption(Options.LIST_SNAPSHOTS.getLongOpt());
563    printSizeInBytes = cmd.hasOption(Options.SIZE_IN_BYTES.getLongOpt());
564    if (cmd.hasOption(Options.REMOTE_DIR.getLongOpt())) {
565      remoteDir = new Path(cmd.getOptionValue(Options.REMOTE_DIR.getLongOpt()));
566    }
567  }
568
569  @Override
570  protected void printUsage() {
571    printUsage("hbase snapshot info [options]", "Options:", "");
572    System.err.println("Examples:");
573    System.err.println("  hbase snapshot info --snapshot MySnapshot --files");
574  }
575
576  /**
577   * Returns the snapshot stats
578   * @param conf     the {@link Configuration} to use
579   * @param snapshot {@link SnapshotDescription} to get stats from
580   * @return the snapshot stats
581   */
582  public static SnapshotStats getSnapshotStats(final Configuration conf,
583    final SnapshotDescription snapshot) throws IOException {
584    SnapshotProtos.SnapshotDescription snapshotDesc =
585      ProtobufUtil.createHBaseProtosSnapshotDesc(snapshot);
586    return getSnapshotStats(conf, snapshotDesc, null);
587  }
588
589  /**
590   * Returns the snapshot stats
591   * @param conf         the {@link Configuration} to use
592   * @param snapshotDesc HBaseProtos.SnapshotDescription to get stats from
593   * @param filesMap     {@link Map} store files map for all snapshots, it may be null
594   * @return the snapshot stats
595   */
596  public static SnapshotStats getSnapshotStats(final Configuration conf,
597    final SnapshotProtos.SnapshotDescription snapshotDesc, final Map<Path, Integer> filesMap)
598    throws IOException {
599    Path rootDir = CommonFSUtils.getRootDir(conf);
600    FileSystem fs = FileSystem.get(rootDir.toUri(), conf);
601    Path snapshotDir = SnapshotDescriptionUtils.getCompletedSnapshotDir(snapshotDesc, rootDir);
602    SnapshotManifest manifest = SnapshotManifest.open(conf, fs, snapshotDir, snapshotDesc);
603    final SnapshotStats stats = new SnapshotStats(conf, fs, snapshotDesc);
604    SnapshotReferenceUtil.concurrentVisitReferencedFiles(conf, fs, manifest,
605      "SnapshotsStatsAggregation", new SnapshotReferenceUtil.SnapshotVisitor() {
606        @Override
607        public void storeFile(final RegionInfo regionInfo, final String family,
608          final SnapshotRegionManifest.StoreFile storeFile) throws IOException {
609          if (!storeFile.hasReference()) {
610            stats.addStoreFile(regionInfo, family, storeFile, filesMap);
611          }
612        }
613      });
614    return stats;
615  }
616
617  /**
618   * Returns the list of available snapshots in the specified location
619   * @param conf the {@link Configuration} to use
620   * @return the list of snapshots
621   */
622  public static List<SnapshotDescription> getSnapshotList(final Configuration conf)
623    throws IOException {
624    Path rootDir = CommonFSUtils.getRootDir(conf);
625    FileSystem fs = FileSystem.get(rootDir.toUri(), conf);
626    Path snapshotDir = SnapshotDescriptionUtils.getSnapshotsDir(rootDir);
627    FileStatus[] snapshots = fs.listStatus(snapshotDir,
628      new SnapshotDescriptionUtils.CompletedSnaphotDirectoriesFilter(fs));
629    List<SnapshotDescription> snapshotLists = new ArrayList<>(snapshots.length);
630    for (FileStatus snapshotDirStat : snapshots) {
631      SnapshotProtos.SnapshotDescription snapshotDesc =
632        SnapshotDescriptionUtils.readSnapshotInfo(fs, snapshotDirStat.getPath());
633      snapshotLists.add(ProtobufUtil.createSnapshotDesc(snapshotDesc));
634    }
635    return snapshotLists;
636  }
637
638  /**
639   * Gets the store files map for snapshot
640   * @param conf                    the {@link Configuration} to use
641   * @param snapshot                {@link SnapshotDescription} to get stats from
642   * @param exec                    the {@link ExecutorService} to use
643   * @param filesMap                {@link Map} the map to put the mapping entries
644   * @param uniqueHFilesArchiveSize {@link AtomicLong} the accumulated store file size in archive
645   * @param uniqueHFilesSize        {@link AtomicLong} the accumulated store file size shared
646   * @param uniqueHFilesMobSize     {@link AtomicLong} the accumulated mob store file size shared
647   */
648  private static void getSnapshotFilesMap(final Configuration conf,
649    final SnapshotDescription snapshot, final ExecutorService exec,
650    final ConcurrentHashMap<Path, Integer> filesMap, final AtomicLong uniqueHFilesArchiveSize,
651    final AtomicLong uniqueHFilesSize, final AtomicLong uniqueHFilesMobSize) throws IOException {
652    SnapshotProtos.SnapshotDescription snapshotDesc =
653      ProtobufUtil.createHBaseProtosSnapshotDesc(snapshot);
654    Path rootDir = CommonFSUtils.getRootDir(conf);
655    final FileSystem fs = FileSystem.get(rootDir.toUri(), conf);
656
657    Path snapshotDir = SnapshotDescriptionUtils.getCompletedSnapshotDir(snapshotDesc, rootDir);
658    SnapshotManifest manifest = SnapshotManifest.open(conf, fs, snapshotDir, snapshotDesc);
659    SnapshotReferenceUtil.concurrentVisitReferencedFiles(conf, fs, manifest, exec,
660      new SnapshotReferenceUtil.SnapshotVisitor() {
661        @Override
662        public void storeFile(final RegionInfo regionInfo, final String family,
663          final SnapshotRegionManifest.StoreFile storeFile) throws IOException {
664          if (!storeFile.hasReference()) {
665            HFileLink link = HFileLink.build(conf, snapshot.getTableName(),
666              regionInfo.getEncodedName(), family, storeFile.getName());
667            long size;
668            Integer count;
669            Path p;
670            AtomicLong al;
671            int c = 0;
672
673            if (fs.exists(link.getArchivePath())) {
674              p = link.getArchivePath();
675              al = uniqueHFilesArchiveSize;
676              size = fs.getFileStatus(p).getLen();
677            } else if (fs.exists(link.getMobPath())) {
678              p = link.getMobPath();
679              al = uniqueHFilesMobSize;
680              size = fs.getFileStatus(p).getLen();
681            } else {
682              p = link.getOriginPath();
683              al = uniqueHFilesSize;
684              size = link.getFileStatus(fs).getLen();
685            }
686
687            // If it has been counted, do not double count
688            count = filesMap.get(p);
689            if (count != null) {
690              c = count.intValue();
691            } else {
692              al.addAndGet(size);
693            }
694
695            filesMap.put(p, ++c);
696          }
697        }
698      });
699  }
700
701  /**
702   * Returns the map of store files based on path for all snapshots
703   * @param conf                    the {@link Configuration} to use
704   * @param uniqueHFilesArchiveSize pass out the size for store files in archive
705   * @param uniqueHFilesSize        pass out the size for store files shared
706   * @param uniqueHFilesMobSize     pass out the size for mob store files shared
707   * @return the map of store files
708   */
709  public static Map<Path, Integer> getSnapshotsFilesMap(final Configuration conf,
710    AtomicLong uniqueHFilesArchiveSize, AtomicLong uniqueHFilesSize, AtomicLong uniqueHFilesMobSize)
711    throws IOException {
712    List<SnapshotDescription> snapshotList = getSnapshotList(conf);
713
714    if (snapshotList.isEmpty()) {
715      return Collections.emptyMap();
716    }
717
718    ConcurrentHashMap<Path, Integer> fileMap = new ConcurrentHashMap<>();
719
720    ExecutorService exec = SnapshotManifest.createExecutor(conf, "SnapshotsFilesMapping");
721
722    try {
723      for (final SnapshotDescription snapshot : snapshotList) {
724        getSnapshotFilesMap(conf, snapshot, exec, fileMap, uniqueHFilesArchiveSize,
725          uniqueHFilesSize, uniqueHFilesMobSize);
726      }
727    } finally {
728      exec.shutdown();
729    }
730
731    return fileMap;
732  }
733
734  public static void main(String[] args) {
735    new SnapshotInfo().doStaticMain(args);
736  }
737}