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.hbase.util.Strings;
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 getStoreFilesSize() == 0
257        ? 0
258        : ((float) hfilesSize.get() / (getStoreFilesSize())) * 100;
259    }
260
261    /** Returns the percentage of the mob store files */
262    public float getMobStoreFilePercentage() {
263      return getStoreFilesSize() == 0
264        ? 0
265        : ((float) hfilesMobSize.get() / (getStoreFilesSize())) * 100;
266    }
267
268    /** Returns the total log size */
269    public long getLogsSize() {
270      return logSize.get();
271    }
272
273    /**
274     * Check if for a give file in archive, if there are other snapshots/tables still reference it.
275     * @param filePath         file path in archive
276     * @param snapshotFilesMap a map for store files in snapshots about how many snapshots refer to
277     *                         it.
278     * @return true or false
279     */
280    private boolean isArchivedFileStillReferenced(final Path filePath,
281      final Map<Path, Integer> snapshotFilesMap) {
282
283      Integer c = snapshotFilesMap.get(filePath);
284
285      // Check if there are other snapshots or table from clone_snapshot() (via back-reference)
286      // still reference to it.
287      if ((c != null) && (c == 1)) {
288        Path parentDir = filePath.getParent();
289        Path backRefDir = HFileLink.getBackReferencesDir(parentDir, filePath.getName());
290        try {
291          if (CommonFSUtils.listStatus(fs, backRefDir) == null) {
292            return false;
293          }
294        } catch (IOException e) {
295          // For the purpose of this function, IOException is ignored and treated as
296          // the file is still being referenced.
297        }
298      }
299      return true;
300    }
301
302    /**
303     * Add the specified store file to the stats
304     * @param region    region encoded Name
305     * @param family    family name
306     * @param storeFile store file name
307     * @param filesMap  store files map for all snapshots, it may be null
308     * @return the store file information
309     */
310    FileInfo addStoreFile(final RegionInfo region, final String family,
311      final SnapshotRegionManifest.StoreFile storeFile, final Map<Path, Integer> filesMap)
312      throws IOException {
313      HFileLink link =
314        HFileLink.build(conf, snapshotTable, region.getEncodedName(), family, storeFile.getName());
315      boolean isCorrupted = false;
316      boolean inArchive = false;
317      long size = -1;
318      try {
319        if (fs.exists(link.getArchivePath())) {
320          inArchive = true;
321          size = fs.getFileStatus(link.getArchivePath()).getLen();
322          hfilesArchiveSize.addAndGet(size);
323          hfilesArchiveCount.incrementAndGet();
324
325          // If store file is not shared with other snapshots and tables,
326          // increase nonSharedHfilesArchiveSize
327          if (
328            (filesMap != null) && !isArchivedFileStillReferenced(link.getArchivePath(), filesMap)
329          ) {
330            nonSharedHfilesArchiveSize.addAndGet(size);
331          }
332        } else if (fs.exists(link.getMobPath())) {
333          inArchive = true;
334          size = fs.getFileStatus(link.getMobPath()).getLen();
335          hfilesMobSize.addAndGet(size);
336          hfilesMobCount.incrementAndGet();
337        } else {
338          size = link.getFileStatus(fs).getLen();
339          hfilesSize.addAndGet(size);
340          hfilesCount.incrementAndGet();
341        }
342        isCorrupted = (storeFile.hasFileSize() && storeFile.getFileSize() != size);
343        if (isCorrupted) hfilesCorrupted.incrementAndGet();
344      } catch (FileNotFoundException e) {
345        hfilesMissing.incrementAndGet();
346      }
347      return new FileInfo(inArchive, size, isCorrupted);
348    }
349
350    /**
351     * Add the specified log file to the stats
352     * @param server  server name
353     * @param logfile log file name
354     * @return the log information
355     */
356    FileInfo addLogFile(final String server, final String logfile) throws IOException {
357      WALLink logLink = new WALLink(conf, server, logfile);
358      long size = -1;
359      try {
360        size = logLink.getFileStatus(fs).getLen();
361        logSize.addAndGet(size);
362        logsCount.incrementAndGet();
363      } catch (FileNotFoundException e) {
364        logsMissing.incrementAndGet();
365      }
366      return new FileInfo(false, size, false);
367    }
368  }
369
370  private FileSystem fs;
371  private Path rootDir;
372
373  private SnapshotManifest snapshotManifest;
374
375  private boolean listSnapshots = false;
376  private String snapshotName;
377  private Path remoteDir;
378  private boolean showSchema = false;
379  private boolean showFiles = false;
380  private boolean showStats = false;
381  private boolean printSizeInBytes = false;
382
383  @Override
384  public int doWork() throws IOException, InterruptedException {
385    if (remoteDir != null) {
386      URI defaultFs = remoteDir.getFileSystem(conf).getUri();
387      CommonFSUtils.setFsDefault(conf, new Path(defaultFs));
388      CommonFSUtils.setRootDir(conf, remoteDir);
389    }
390
391    // List Available Snapshots
392    if (listSnapshots) {
393      SimpleDateFormat df = new SimpleDateFormat("yyyy-MM-dd'T'HH:mm:ss");
394      System.out.printf("%-20s | %-20s | %-20s | %s%n", "SNAPSHOT", "CREATION TIME", "TTL IN SEC",
395        "TABLE NAME");
396      for (SnapshotDescription desc : getSnapshotList(conf)) {
397        System.out.printf("%-20s | %20s | %20s | %s%n", desc.getName(),
398          df.format(new Date(desc.getCreationTime())), desc.getTtl(), desc.getTableNameAsString());
399      }
400      return 0;
401    }
402
403    rootDir = CommonFSUtils.getRootDir(conf);
404    fs = FileSystem.get(rootDir.toUri(), conf);
405    LOG.debug("fs=" + fs.getUri().toString() + " root=" + rootDir);
406
407    // Load snapshot information
408    if (!loadSnapshotInfo(snapshotName)) {
409      System.err.println("Snapshot '" + snapshotName + "' not found!");
410      return 1;
411    }
412
413    printInfo();
414    if (showSchema) {
415      printSchema();
416    }
417    printFiles(showFiles, showStats);
418
419    return 0;
420  }
421
422  /**
423   * Load snapshot info and table descriptor for the specified snapshot
424   * @param snapshotName name of the snapshot to load
425   * @return false if snapshot is not found
426   */
427  private boolean loadSnapshotInfo(final String snapshotName) throws IOException {
428    Path snapshotDir = SnapshotDescriptionUtils.getCompletedSnapshotDir(snapshotName, rootDir);
429    if (!fs.exists(snapshotDir)) {
430      LOG.warn("Snapshot '" + snapshotName + "' not found in: " + snapshotDir);
431      return false;
432    }
433
434    SnapshotProtos.SnapshotDescription snapshotDesc =
435      SnapshotDescriptionUtils.readSnapshotInfo(fs, snapshotDir);
436    snapshotManifest = SnapshotManifest.open(getConf(), fs, snapshotDir, snapshotDesc);
437    return true;
438  }
439
440  /**
441   * Dump the {@link SnapshotDescription}
442   */
443  private void printInfo() {
444    SnapshotProtos.SnapshotDescription snapshotDesc = snapshotManifest.getSnapshotDescription();
445    SimpleDateFormat df = new SimpleDateFormat("yyyy-MM-dd'T'HH:mm:ss");
446    System.out.println("Snapshot Info");
447    System.out.println("----------------------------------------");
448    System.out.println("   Name: " + snapshotDesc.getName());
449    System.out.println("   Type: " + snapshotDesc.getType());
450    System.out.println("  Table: " + snapshotDesc.getTable());
451    System.out.println(" Format: " + snapshotDesc.getVersion());
452    System.out.println("Created: " + df.format(new Date(snapshotDesc.getCreationTime())));
453    System.out.println("    Ttl: " + snapshotDesc.getTtl());
454    System.out.println("  Owner: " + snapshotDesc.getOwner());
455    System.out.println();
456  }
457
458  /**
459   * Dump the {@link org.apache.hadoop.hbase.client.TableDescriptor}
460   */
461  private void printSchema() {
462    System.out.println("Table Descriptor");
463    System.out.println("----------------------------------------");
464    System.out.println(snapshotManifest.getTableDescriptor().toString());
465    System.out.println();
466  }
467
468  /**
469   * Collect the hfiles and logs statistics of the snapshot and dump the file list if requested and
470   * the collected information.
471   */
472  private void printFiles(final boolean showFiles, final boolean showStats) throws IOException {
473    if (showFiles) {
474      System.out.println("Snapshot Files");
475      System.out.println("----------------------------------------");
476    }
477
478    // Collect information about hfiles and logs in the snapshot
479    final SnapshotProtos.SnapshotDescription snapshotDesc =
480      snapshotManifest.getSnapshotDescription();
481    final String table = snapshotDesc.getTable();
482    final SnapshotDescription desc = ProtobufUtil.createSnapshotDesc(snapshotDesc);
483    final SnapshotStats stats = new SnapshotStats(this.getConf(), this.fs, desc);
484    SnapshotReferenceUtil.concurrentVisitReferencedFiles(getConf(), fs, snapshotManifest,
485      "SnapshotInfo", new SnapshotReferenceUtil.SnapshotVisitor() {
486        @Override
487        public void storeFile(final RegionInfo regionInfo, final String family,
488          final SnapshotRegionManifest.StoreFile storeFile) throws IOException {
489          if (storeFile.hasReference()) return;
490
491          SnapshotStats.FileInfo info = stats.addStoreFile(regionInfo, family, storeFile, null);
492          if (showFiles) {
493            String state = info.getStateToString();
494            System.out.printf("%8s %s/%s/%s/%s %s%n",
495              (info.isMissing() ? "-" : fileSizeToString(info.getSize())), table,
496              regionInfo.getEncodedName(), family, storeFile.getName(),
497              state == null ? "" : "(" + state + ")");
498          }
499        }
500      });
501
502    // Dump the stats
503    System.out.println();
504    if (stats.isSnapshotCorrupted()) {
505      System.out.println("**************************************************************");
506      System.out.printf("BAD SNAPSHOT: %d hfile(s) and %d log(s) missing.%n",
507        stats.getMissingStoreFilesCount(), stats.getMissingLogsCount());
508      System.out.printf("              %d hfile(s) corrupted.%n",
509        stats.getCorruptedStoreFilesCount());
510      System.out.println("**************************************************************");
511    }
512
513    if (showStats) {
514      System.out.printf(
515        "%d HFiles (%d in archive, %d in mob storage), total size %s "
516          + "(%.2f%% %s shared with the source table, %.2f%% %s in mob dir)%n",
517        stats.getStoreFilesCount(), stats.getArchivedStoreFilesCount(),
518        stats.getMobStoreFilesCount(), fileSizeToString(stats.getStoreFilesSize()),
519        stats.getSharedStoreFilePercentage(), fileSizeToString(stats.getSharedStoreFilesSize()),
520        stats.getMobStoreFilePercentage(), fileSizeToString(stats.getMobStoreFilesSize()));
521      System.out.printf("%d Logs, total size %s%n", stats.getLogsCount(),
522        fileSizeToString(stats.getLogsSize()));
523      System.out.println();
524    }
525  }
526
527  private String fileSizeToString(long size) {
528    return printSizeInBytes ? Long.toString(size) : Strings.humanReadableInt(size);
529  }
530
531  @Override
532  protected void addOptions() {
533    addOption(Options.SNAPSHOT);
534    addOption(Options.REMOTE_DIR);
535    addOption(Options.LIST_SNAPSHOTS);
536    addOption(Options.FILES);
537    addOption(Options.STATS);
538    addOption(Options.SCHEMA);
539    addOption(Options.SIZE_IN_BYTES);
540  }
541
542  @Override
543  protected CommandLineParser newParser() {
544    // Commons-CLI lacks the capability to handle combinations of options, so we do it ourselves
545    // Validate in parse() to get helpful error messages instead of exploding in processOptions()
546    return new DefaultParser() {
547      @Override
548      public CommandLine parse(org.apache.hbase.thirdparty.org.apache.commons.cli.Options opts,
549        String[] args, Properties props, boolean stop) throws ParseException {
550        CommandLine cl = super.parse(opts, args, props, stop);
551        if (!cmd.hasOption(Options.LIST_SNAPSHOTS) && !cmd.hasOption(Options.SNAPSHOT)) {
552          throw new ParseException("Missing required snapshot option!");
553        }
554        return cl;
555      }
556    };
557  }
558
559  @Override
560  protected void processOptions(CommandLine cmd) {
561    snapshotName = cmd.getOptionValue(Options.SNAPSHOT.getLongOpt());
562    showFiles = cmd.hasOption(Options.FILES.getLongOpt());
563    showStats =
564      cmd.hasOption(Options.FILES.getLongOpt()) || cmd.hasOption(Options.STATS.getLongOpt());
565    showSchema = cmd.hasOption(Options.SCHEMA.getLongOpt());
566    listSnapshots = cmd.hasOption(Options.LIST_SNAPSHOTS.getLongOpt());
567    printSizeInBytes = cmd.hasOption(Options.SIZE_IN_BYTES.getLongOpt());
568    if (cmd.hasOption(Options.REMOTE_DIR.getLongOpt())) {
569      remoteDir = new Path(cmd.getOptionValue(Options.REMOTE_DIR.getLongOpt()));
570    }
571  }
572
573  @Override
574  protected void printUsage() {
575    printUsage("hbase snapshot info [options]", "Options:", "");
576    System.err.println("Examples:");
577    System.err.println("  hbase snapshot info --snapshot MySnapshot --files");
578  }
579
580  /**
581   * Returns the snapshot stats
582   * @param conf     the {@link Configuration} to use
583   * @param snapshot {@link SnapshotDescription} to get stats from
584   * @return the snapshot stats
585   */
586  public static SnapshotStats getSnapshotStats(final Configuration conf,
587    final SnapshotDescription snapshot) throws IOException {
588    SnapshotProtos.SnapshotDescription snapshotDesc =
589      ProtobufUtil.createHBaseProtosSnapshotDesc(snapshot);
590    return getSnapshotStats(conf, snapshotDesc, null);
591  }
592
593  /**
594   * Returns the snapshot stats
595   * @param conf         the {@link Configuration} to use
596   * @param snapshotDesc HBaseProtos.SnapshotDescription to get stats from
597   * @param filesMap     {@link Map} store files map for all snapshots, it may be null
598   * @return the snapshot stats
599   */
600  public static SnapshotStats getSnapshotStats(final Configuration conf,
601    final SnapshotProtos.SnapshotDescription snapshotDesc, final Map<Path, Integer> filesMap)
602    throws IOException {
603    Path rootDir = CommonFSUtils.getRootDir(conf);
604    FileSystem fs = FileSystem.get(rootDir.toUri(), conf);
605    Path snapshotDir = SnapshotDescriptionUtils.getCompletedSnapshotDir(snapshotDesc, rootDir);
606    SnapshotManifest manifest = SnapshotManifest.open(conf, fs, snapshotDir, snapshotDesc);
607    final SnapshotStats stats = new SnapshotStats(conf, fs, snapshotDesc);
608    SnapshotReferenceUtil.concurrentVisitReferencedFiles(conf, fs, manifest,
609      "SnapshotsStatsAggregation", new SnapshotReferenceUtil.SnapshotVisitor() {
610        @Override
611        public void storeFile(final RegionInfo regionInfo, final String family,
612          final SnapshotRegionManifest.StoreFile storeFile) throws IOException {
613          if (!storeFile.hasReference()) {
614            stats.addStoreFile(regionInfo, family, storeFile, filesMap);
615          }
616        }
617      });
618    return stats;
619  }
620
621  /**
622   * Returns the list of available snapshots in the specified location
623   * @param conf the {@link Configuration} to use
624   * @return the list of snapshots
625   */
626  public static List<SnapshotDescription> getSnapshotList(final Configuration conf)
627    throws IOException {
628    Path rootDir = CommonFSUtils.getRootDir(conf);
629    FileSystem fs = FileSystem.get(rootDir.toUri(), conf);
630    Path snapshotDir = SnapshotDescriptionUtils.getSnapshotsDir(rootDir);
631    FileStatus[] snapshots = fs.listStatus(snapshotDir,
632      new SnapshotDescriptionUtils.CompletedSnaphotDirectoriesFilter(fs));
633    List<SnapshotDescription> snapshotLists = new ArrayList<>(snapshots.length);
634    for (FileStatus snapshotDirStat : snapshots) {
635      SnapshotProtos.SnapshotDescription snapshotDesc =
636        SnapshotDescriptionUtils.readSnapshotInfo(fs, snapshotDirStat.getPath());
637      snapshotLists.add(ProtobufUtil.createSnapshotDesc(snapshotDesc));
638    }
639    return snapshotLists;
640  }
641
642  /**
643   * Gets the store files map for snapshot
644   * @param conf                    the {@link Configuration} to use
645   * @param snapshot                {@link SnapshotDescription} to get stats from
646   * @param exec                    the {@link ExecutorService} to use
647   * @param filesMap                {@link Map} the map to put the mapping entries
648   * @param uniqueHFilesArchiveSize {@link AtomicLong} the accumulated store file size in archive
649   * @param uniqueHFilesSize        {@link AtomicLong} the accumulated store file size shared
650   * @param uniqueHFilesMobSize     {@link AtomicLong} the accumulated mob store file size shared
651   */
652  private static void getSnapshotFilesMap(final Configuration conf,
653    final SnapshotDescription snapshot, final ExecutorService exec,
654    final ConcurrentHashMap<Path, Integer> filesMap, final AtomicLong uniqueHFilesArchiveSize,
655    final AtomicLong uniqueHFilesSize, final AtomicLong uniqueHFilesMobSize) throws IOException {
656    SnapshotProtos.SnapshotDescription snapshotDesc =
657      ProtobufUtil.createHBaseProtosSnapshotDesc(snapshot);
658    Path rootDir = CommonFSUtils.getRootDir(conf);
659    final FileSystem fs = FileSystem.get(rootDir.toUri(), conf);
660
661    Path snapshotDir = SnapshotDescriptionUtils.getCompletedSnapshotDir(snapshotDesc, rootDir);
662    SnapshotManifest manifest = SnapshotManifest.open(conf, fs, snapshotDir, snapshotDesc);
663    SnapshotReferenceUtil.concurrentVisitReferencedFiles(conf, fs, manifest, exec,
664      new SnapshotReferenceUtil.SnapshotVisitor() {
665        @Override
666        public void storeFile(final RegionInfo regionInfo, final String family,
667          final SnapshotRegionManifest.StoreFile storeFile) throws IOException {
668          if (!storeFile.hasReference()) {
669            HFileLink link = HFileLink.build(conf, snapshot.getTableName(),
670              regionInfo.getEncodedName(), family, storeFile.getName());
671            long size;
672            Integer count;
673            Path p;
674            AtomicLong al;
675            int c = 0;
676
677            if (fs.exists(link.getArchivePath())) {
678              p = link.getArchivePath();
679              al = uniqueHFilesArchiveSize;
680              size = fs.getFileStatus(p).getLen();
681            } else if (fs.exists(link.getMobPath())) {
682              p = link.getMobPath();
683              al = uniqueHFilesMobSize;
684              size = fs.getFileStatus(p).getLen();
685            } else {
686              p = link.getOriginPath();
687              al = uniqueHFilesSize;
688              size = link.getFileStatus(fs).getLen();
689            }
690
691            // If it has been counted, do not double count
692            count = filesMap.get(p);
693            if (count != null) {
694              c = count.intValue();
695            } else {
696              al.addAndGet(size);
697            }
698
699            filesMap.put(p, ++c);
700          }
701        }
702      });
703  }
704
705  /**
706   * Returns the map of store files based on path for all snapshots
707   * @param conf                    the {@link Configuration} to use
708   * @param uniqueHFilesArchiveSize pass out the size for store files in archive
709   * @param uniqueHFilesSize        pass out the size for store files shared
710   * @param uniqueHFilesMobSize     pass out the size for mob store files shared
711   * @return the map of store files
712   */
713  public static Map<Path, Integer> getSnapshotsFilesMap(final Configuration conf,
714    AtomicLong uniqueHFilesArchiveSize, AtomicLong uniqueHFilesSize, AtomicLong uniqueHFilesMobSize)
715    throws IOException {
716    List<SnapshotDescription> snapshotList = getSnapshotList(conf);
717
718    if (snapshotList.isEmpty()) {
719      return Collections.emptyMap();
720    }
721
722    ConcurrentHashMap<Path, Integer> fileMap = new ConcurrentHashMap<>();
723
724    ExecutorService exec = SnapshotManifest.createExecutor(conf, "SnapshotsFilesMapping");
725
726    try {
727      for (final SnapshotDescription snapshot : snapshotList) {
728        getSnapshotFilesMap(conf, snapshot, exec, fileMap, uniqueHFilesArchiveSize,
729          uniqueHFilesSize, uniqueHFilesMobSize);
730      }
731    } finally {
732      exec.shutdown();
733    }
734
735    return fileMap;
736  }
737
738  public static void main(String[] args) {
739    new SnapshotInfo().doStaticMain(args);
740  }
741}