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