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.IOException;
021import java.io.InputStream;
022import java.io.OutputStream;
023import java.util.ArrayList;
024import java.util.Arrays;
025import java.util.Collections;
026import java.util.HashMap;
027import java.util.HashSet;
028import java.util.LinkedList;
029import java.util.List;
030import java.util.Map;
031import java.util.Map.Entry;
032import java.util.Set;
033import java.util.TreeMap;
034import java.util.concurrent.ThreadPoolExecutor;
035import org.apache.hadoop.conf.Configuration;
036import org.apache.hadoop.fs.FileStatus;
037import org.apache.hadoop.fs.FileSystem;
038import org.apache.hadoop.fs.Path;
039import org.apache.hadoop.hbase.MetaTableAccessor;
040import org.apache.hadoop.hbase.TableName;
041import org.apache.hadoop.hbase.backup.HFileArchiver;
042import org.apache.hadoop.hbase.client.Connection;
043import org.apache.hadoop.hbase.client.ConnectionFactory;
044import org.apache.hadoop.hbase.client.RegionInfo;
045import org.apache.hadoop.hbase.client.RegionInfoBuilder;
046import org.apache.hadoop.hbase.client.TableDescriptor;
047import org.apache.hadoop.hbase.errorhandling.ForeignExceptionDispatcher;
048import org.apache.hadoop.hbase.io.HFileLink;
049import org.apache.hadoop.hbase.io.Reference;
050import org.apache.hadoop.hbase.mob.MobUtils;
051import org.apache.hadoop.hbase.monitoring.MonitoredTask;
052import org.apache.hadoop.hbase.monitoring.TaskMonitor;
053import org.apache.hadoop.hbase.regionserver.HRegion;
054import org.apache.hadoop.hbase.regionserver.HRegionFileSystem;
055import org.apache.hadoop.hbase.regionserver.StoreContext;
056import org.apache.hadoop.hbase.regionserver.StoreFileInfo;
057import org.apache.hadoop.hbase.regionserver.StoreUtils;
058import org.apache.hadoop.hbase.regionserver.storefiletracker.StoreFileTracker;
059import org.apache.hadoop.hbase.regionserver.storefiletracker.StoreFileTrackerFactory;
060import org.apache.hadoop.hbase.security.access.AccessControlClient;
061import org.apache.hadoop.hbase.security.access.Permission;
062import org.apache.hadoop.hbase.security.access.ShadedAccessControlUtil;
063import org.apache.hadoop.hbase.security.access.TablePermission;
064import org.apache.hadoop.hbase.snapshot.RestoreSnapshotHelper.RestoreMetaChanges;
065import org.apache.hadoop.hbase.util.Bytes;
066import org.apache.hadoop.hbase.util.CommonFSUtils;
067import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
068import org.apache.hadoop.hbase.util.FSUtils;
069import org.apache.hadoop.hbase.util.ModifyRegionUtils;
070import org.apache.hadoop.hbase.util.Pair;
071import org.apache.hadoop.io.IOUtils;
072import org.apache.yetus.audience.InterfaceAudience;
073import org.slf4j.Logger;
074import org.slf4j.LoggerFactory;
075
076import org.apache.hbase.thirdparty.com.google.common.collect.ListMultimap;
077
078import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
079import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription;
080import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotRegionManifest;
081
082/**
083 * Helper to Restore/Clone a Snapshot
084 * <p>
085 * The helper assumes that a table is already created, and by calling restore() the content present
086 * in the snapshot will be restored as the new content of the table.
087 * <p>
088 * Clone from Snapshot: If the target table is empty, the restore operation is just a "clone
089 * operation", where the only operations are:
090 * <ul>
091 * <li>for each region in the snapshot create a new region (note that the region will have a
092 * different name, since the encoding contains the table name)
093 * <li>for each file in the region create a new HFileLink to point to the original file.
094 * <li>restore the logs, if any
095 * </ul>
096 * <p>
097 * Restore from Snapshot:
098 * <ul>
099 * <li>for each region in the table verify which are available in the snapshot and which are not
100 * <ul>
101 * <li>if the region is not present in the snapshot, remove it.
102 * <li>if the region is present in the snapshot
103 * <ul>
104 * <li>for each file in the table region verify which are available in the snapshot
105 * <ul>
106 * <li>if the hfile is not present in the snapshot, remove it
107 * <li>if the hfile is present, keep it (nothing to do)
108 * </ul>
109 * <li>for each file in the snapshot region but not in the table
110 * <ul>
111 * <li>create a new HFileLink that point to the original file
112 * </ul>
113 * </ul>
114 * </ul>
115 * <li>for each region in the snapshot not present in the current table state
116 * <ul>
117 * <li>create a new region and for each file in the region create a new HFileLink (This is the same
118 * as the clone operation)
119 * </ul>
120 * <li>restore the logs, if any
121 * </ul>
122 */
123@InterfaceAudience.Private
124public class RestoreSnapshotHelper {
125  private static final Logger LOG = LoggerFactory.getLogger(RestoreSnapshotHelper.class);
126
127  private final Map<byte[], byte[]> regionsMap = new TreeMap<>(Bytes.BYTES_COMPARATOR);
128
129  private final Map<String, Pair<String, String>> parentsMap = new HashMap<>();
130
131  private final ForeignExceptionDispatcher monitor;
132  private final MonitoredTask status;
133
134  private final SnapshotManifest snapshotManifest;
135  private final SnapshotDescription snapshotDesc;
136  private final TableName snapshotTable;
137
138  private final TableDescriptor tableDesc;
139  private final Path rootDir;
140  private final Path tableDir;
141
142  private final Configuration conf;
143  private final FileSystem fs;
144  private final boolean createBackRefs;
145
146  public RestoreSnapshotHelper(final Configuration conf, final FileSystem fs,
147    final SnapshotManifest manifest, final TableDescriptor tableDescriptor, final Path rootDir,
148    final ForeignExceptionDispatcher monitor, final MonitoredTask status) {
149    this(conf, fs, manifest, tableDescriptor, rootDir, monitor, status, true);
150  }
151
152  public RestoreSnapshotHelper(final Configuration conf, final FileSystem fs,
153    final SnapshotManifest manifest, final TableDescriptor tableDescriptor, final Path rootDir,
154    final ForeignExceptionDispatcher monitor, final MonitoredTask status,
155    final boolean createBackRefs) {
156    this.fs = fs;
157    this.conf = conf;
158    this.snapshotManifest = manifest;
159    this.snapshotDesc = manifest.getSnapshotDescription();
160    this.snapshotTable = TableName.valueOf(snapshotDesc.getTable());
161    this.tableDesc = tableDescriptor;
162    this.rootDir = rootDir;
163    this.tableDir = CommonFSUtils.getTableDir(rootDir, tableDesc.getTableName());
164    this.monitor = monitor;
165    this.status = status;
166    this.createBackRefs = createBackRefs;
167  }
168
169  /**
170   * Restore the on-disk table to a specified snapshot state.
171   * @return the set of regions touched by the restore operation
172   */
173  public RestoreMetaChanges restoreHdfsRegions() throws IOException {
174    ThreadPoolExecutor exec = SnapshotManifest.createExecutor(conf, "RestoreSnapshot");
175    try {
176      return restoreHdfsRegions(exec);
177    } finally {
178      exec.shutdown();
179    }
180  }
181
182  private RestoreMetaChanges restoreHdfsRegions(final ThreadPoolExecutor exec) throws IOException {
183    LOG.info("starting restore table regions using snapshot=" + snapshotDesc);
184
185    Map<String, SnapshotRegionManifest> regionManifests = snapshotManifest.getRegionManifestsMap();
186    if (regionManifests == null) {
187      LOG.warn("Nothing to restore. Snapshot " + snapshotDesc + " looks empty");
188      return null;
189    }
190
191    RestoreMetaChanges metaChanges = new RestoreMetaChanges(tableDesc, parentsMap);
192
193    // Take a copy of the manifest.keySet() since we are going to modify
194    // this instance, by removing the regions already present in the restore dir.
195    Set<String> regionNames = new HashSet<>(regionManifests.keySet());
196
197    List<RegionInfo> tableRegions = getTableRegions();
198
199    RegionInfo mobRegion =
200      MobUtils.getMobRegionInfo(snapshotManifest.getTableDescriptor().getTableName());
201    if (tableRegions != null) {
202      // restore the mob region in case
203      if (regionNames.contains(mobRegion.getEncodedName())) {
204        monitor.rethrowException();
205        status.setStatus("Restoring mob region...");
206        List<RegionInfo> mobRegions = new ArrayList<>(1);
207        mobRegions.add(mobRegion);
208        restoreHdfsMobRegions(exec, regionManifests, mobRegions);
209        regionNames.remove(mobRegion.getEncodedName());
210        status.setStatus("Finished restoring mob region.");
211      }
212    }
213    if (regionNames.contains(mobRegion.getEncodedName())) {
214      // add the mob region
215      monitor.rethrowException();
216      status.setStatus("Cloning mob region...");
217      cloneHdfsMobRegion(regionManifests, mobRegion);
218      regionNames.remove(mobRegion.getEncodedName());
219      status.setStatus("Finished cloning mob region.");
220    }
221
222    // Identify which region are still available and which not.
223    // NOTE: we rely upon the region name as: "table name, start key, end key"
224    if (tableRegions != null) {
225      monitor.rethrowException();
226      for (RegionInfo regionInfo : tableRegions) {
227        String regionName = regionInfo.getEncodedName();
228        if (regionNames.contains(regionName)) {
229          LOG.info("region to restore: " + regionName);
230          regionNames.remove(regionName);
231          metaChanges.addRegionToRestore(
232            ProtobufUtil.toRegionInfo(regionManifests.get(regionName).getRegionInfo()));
233        } else {
234          LOG.info("region to remove: " + regionName);
235          metaChanges.addRegionToRemove(regionInfo);
236        }
237      }
238    }
239
240    // Regions to Add: present in the snapshot but not in the current table
241    List<RegionInfo> regionsToAdd = new ArrayList<>(regionNames.size());
242    if (regionNames.size() > 0) {
243      monitor.rethrowException();
244      for (String regionName : regionNames) {
245        LOG.info("region to add: " + regionName);
246        regionsToAdd
247          .add(ProtobufUtil.toRegionInfo(regionManifests.get(regionName).getRegionInfo()));
248      }
249    }
250
251    // Create new regions cloning from the snapshot
252    // HBASE-19980: We need to call cloneHdfsRegions() before restoreHdfsRegions() because
253    // regionsMap is constructed in cloneHdfsRegions() and it can be used in restoreHdfsRegions().
254    monitor.rethrowException();
255    status.setStatus("Cloning regions...");
256    RegionInfo[] clonedRegions = cloneHdfsRegions(exec, regionManifests, regionsToAdd);
257    metaChanges.setNewRegions(clonedRegions);
258    status.setStatus("Finished cloning regions.");
259
260    // Restore regions using the snapshot data
261    monitor.rethrowException();
262    status.setStatus("Restoring table regions...");
263    restoreHdfsRegions(exec, regionManifests, metaChanges.getRegionsToRestore());
264    status.setStatus("Finished restoring all table regions.");
265
266    // Remove regions from the current table
267    monitor.rethrowException();
268    status.setStatus("Starting to delete excess regions from table");
269    removeHdfsRegions(exec, metaChanges.getRegionsToRemove());
270    status.setStatus("Finished deleting excess regions from table.");
271
272    LOG.info("finishing restore table regions using snapshot=" + snapshotDesc);
273
274    return metaChanges;
275  }
276
277  /**
278   * Describe the set of operations needed to update hbase:meta after restore.
279   */
280  public static class RestoreMetaChanges {
281    private final Map<String, Pair<String, String>> parentsMap;
282    private final TableDescriptor htd;
283
284    private List<RegionInfo> regionsToRestore = null;
285    private List<RegionInfo> regionsToRemove = null;
286    private List<RegionInfo> regionsToAdd = null;
287
288    public RestoreMetaChanges(TableDescriptor htd, Map<String, Pair<String, String>> parentsMap) {
289      this.parentsMap = parentsMap;
290      this.htd = htd;
291    }
292
293    public TableDescriptor getTableDescriptor() {
294      return htd;
295    }
296
297    /**
298     * Returns the map of parent-children_pair.
299     * @return the map
300     */
301    public Map<String, Pair<String, String>> getParentToChildrenPairMap() {
302      return this.parentsMap;
303    }
304
305    /** Returns true if there're new regions */
306    public boolean hasRegionsToAdd() {
307      return this.regionsToAdd != null && this.regionsToAdd.size() > 0;
308    }
309
310    /**
311     * Returns the list of new regions added during the on-disk restore. The caller is responsible
312     * to add the regions to META. e.g MetaTableAccessor.addRegionsToMeta(...)
313     * @return the list of regions to add to META
314     */
315    public List<RegionInfo> getRegionsToAdd() {
316      return this.regionsToAdd;
317    }
318
319    /** Returns true if there're regions to restore */
320    public boolean hasRegionsToRestore() {
321      return this.regionsToRestore != null && this.regionsToRestore.size() > 0;
322    }
323
324    /**
325     * Returns the list of 'restored regions' during the on-disk restore. The caller is responsible
326     * to add the regions to hbase:meta if not present.
327     * @return the list of regions restored
328     */
329    public List<RegionInfo> getRegionsToRestore() {
330      return this.regionsToRestore;
331    }
332
333    /** Returns true if there're regions to remove */
334    public boolean hasRegionsToRemove() {
335      return this.regionsToRemove != null && this.regionsToRemove.size() > 0;
336    }
337
338    /**
339     * Returns the list of regions removed during the on-disk restore. The caller is responsible to
340     * remove the regions from META. e.g. MetaTableAccessor.deleteRegions(...)
341     * @return the list of regions to remove from META
342     */
343    public List<RegionInfo> getRegionsToRemove() {
344      return this.regionsToRemove;
345    }
346
347    void setNewRegions(final RegionInfo[] hris) {
348      if (hris != null) {
349        regionsToAdd = Arrays.asList(hris);
350      } else {
351        regionsToAdd = null;
352      }
353    }
354
355    void addRegionToRemove(final RegionInfo hri) {
356      if (regionsToRemove == null) {
357        regionsToRemove = new LinkedList<>();
358      }
359      regionsToRemove.add(hri);
360    }
361
362    void addRegionToRestore(final RegionInfo hri) {
363      if (regionsToRestore == null) {
364        regionsToRestore = new LinkedList<>();
365      }
366      regionsToRestore.add(hri);
367    }
368
369    public void updateMetaParentRegions(Connection connection, final List<RegionInfo> regionInfos)
370      throws IOException {
371      if (regionInfos == null || parentsMap.isEmpty()) return;
372
373      // Extract region names and offlined regions
374      Map<String, RegionInfo> regionsByName = new HashMap<>(regionInfos.size());
375      List<RegionInfo> parentRegions = new LinkedList<>();
376      for (RegionInfo regionInfo : regionInfos) {
377        if (regionInfo.isSplitParent()) {
378          parentRegions.add(regionInfo);
379        } else {
380          regionsByName.put(regionInfo.getEncodedName(), regionInfo);
381        }
382      }
383
384      // Update Offline parents
385      for (RegionInfo regionInfo : parentRegions) {
386        Pair<String, String> daughters = parentsMap.get(regionInfo.getEncodedName());
387        if (daughters == null) {
388          // The snapshot contains an unreferenced region.
389          // It will be removed by the CatalogJanitor.
390          LOG.warn("Skip update of unreferenced offline parent: " + regionInfo);
391          continue;
392        }
393
394        // One side of the split is already compacted
395        if (daughters.getSecond() == null) {
396          daughters.setSecond(daughters.getFirst());
397        }
398
399        LOG.debug("Update splits parent " + regionInfo.getEncodedName() + " -> " + daughters);
400        MetaTableAccessor.addSplitsToParent(connection, regionInfo,
401          regionsByName.get(daughters.getFirst()), regionsByName.get(daughters.getSecond()));
402      }
403    }
404  }
405
406  /**
407   * Remove specified regions from the file-system, using the archiver.
408   */
409  private void removeHdfsRegions(final ThreadPoolExecutor exec, final List<RegionInfo> regions)
410    throws IOException {
411    if (regions == null || regions.isEmpty()) return;
412    ModifyRegionUtils.editRegions(exec, regions, new ModifyRegionUtils.RegionEditTask() {
413      @Override
414      public void editRegion(final RegionInfo hri) throws IOException {
415        HFileArchiver.archiveRegion(conf, fs, hri);
416      }
417    });
418  }
419
420  /**
421   * Restore specified regions by restoring content to the snapshot state.
422   */
423  private void restoreHdfsRegions(final ThreadPoolExecutor exec,
424    final Map<String, SnapshotRegionManifest> regionManifests, final List<RegionInfo> regions)
425    throws IOException {
426    if (regions == null || regions.isEmpty()) return;
427    ModifyRegionUtils.editRegions(exec, regions, new ModifyRegionUtils.RegionEditTask() {
428      @Override
429      public void editRegion(final RegionInfo hri) throws IOException {
430        restoreRegion(hri, regionManifests.get(hri.getEncodedName()));
431      }
432    });
433  }
434
435  /**
436   * Restore specified mob regions by restoring content to the snapshot state.
437   */
438  private void restoreHdfsMobRegions(final ThreadPoolExecutor exec,
439    final Map<String, SnapshotRegionManifest> regionManifests, final List<RegionInfo> regions)
440    throws IOException {
441    if (regions == null || regions.isEmpty()) return;
442    ModifyRegionUtils.editRegions(exec, regions, new ModifyRegionUtils.RegionEditTask() {
443      @Override
444      public void editRegion(final RegionInfo hri) throws IOException {
445        restoreMobRegion(hri, regionManifests.get(hri.getEncodedName()));
446      }
447    });
448  }
449
450  private Map<String, List<SnapshotRegionManifest.StoreFile>>
451    getRegionHFileReferences(final SnapshotRegionManifest manifest) {
452    Map<String, List<SnapshotRegionManifest.StoreFile>> familyMap =
453      new HashMap<>(manifest.getFamilyFilesCount());
454    for (SnapshotRegionManifest.FamilyFiles familyFiles : manifest.getFamilyFilesList()) {
455      familyMap.put(familyFiles.getFamilyName().toStringUtf8(),
456        new ArrayList<>(familyFiles.getStoreFilesList()));
457    }
458    return familyMap;
459  }
460
461  /**
462   * Restore region by removing files not in the snapshot and adding the missing ones from the
463   * snapshot.
464   */
465  private void restoreRegion(final RegionInfo regionInfo,
466    final SnapshotRegionManifest regionManifest) throws IOException {
467    restoreRegion(regionInfo, regionManifest, new Path(tableDir, regionInfo.getEncodedName()));
468  }
469
470  /**
471   * Restore mob region by removing files not in the snapshot and adding the missing ones from the
472   * snapshot.
473   */
474  private void restoreMobRegion(final RegionInfo regionInfo,
475    final SnapshotRegionManifest regionManifest) throws IOException {
476    if (regionManifest == null) {
477      return;
478    }
479    restoreRegion(regionInfo, regionManifest,
480      MobUtils.getMobRegionPath(conf, tableDesc.getTableName()));
481  }
482
483  /**
484   * Restore region by removing files not in the snapshot and adding the missing ones from the
485   * snapshot.
486   */
487  private void restoreRegion(final RegionInfo regionInfo,
488    final SnapshotRegionManifest regionManifest, Path regionDir) throws IOException {
489    Map<String, List<SnapshotRegionManifest.StoreFile>> snapshotFiles =
490      getRegionHFileReferences(regionManifest);
491
492    String tableName = tableDesc.getTableName().getNameAsString();
493    final String snapshotName = snapshotDesc.getName();
494
495    Path regionPath = new Path(tableDir, regionInfo.getEncodedName());
496    HRegionFileSystem regionFS = (fs.exists(regionPath))
497      ? HRegionFileSystem.openRegionFromFileSystem(conf, fs, tableDir, regionInfo, false)
498      : HRegionFileSystem.createRegionOnFileSystem(conf, fs, tableDir, regionInfo);
499
500    // Restore families present in the table
501    for (Path familyDir : FSUtils.getFamilyDirs(fs, regionDir)) {
502      byte[] family = Bytes.toBytes(familyDir.getName());
503
504      StoreFileTracker tracker = StoreFileTrackerFactory.create(conf, true,
505        StoreContext.getBuilder().withColumnFamilyDescriptor(tableDesc.getColumnFamily(family))
506          .withFamilyStoreDirectoryPath(familyDir).withRegionFileSystem(regionFS).build());
507      Set<String> familyFiles = getTableRegionFamilyFiles(familyDir);
508      List<SnapshotRegionManifest.StoreFile> snapshotFamilyFiles =
509        snapshotFiles.remove(familyDir.getName());
510      List<StoreFileInfo> filesToTrack = new ArrayList<>();
511      if (snapshotFamilyFiles != null) {
512        List<SnapshotRegionManifest.StoreFile> hfilesToAdd = new ArrayList<>();
513        for (SnapshotRegionManifest.StoreFile storeFile : snapshotFamilyFiles) {
514          if (familyFiles.contains(storeFile.getName())) {
515            // HFile already present
516            familyFiles.remove(storeFile.getName());
517            // no need to restore already present files, but we need to add those to tracker
518            filesToTrack
519              .add(tracker.getStoreFileInfo(new Path(familyDir, storeFile.getName()), true));
520          } else {
521            // HFile missing
522            hfilesToAdd.add(storeFile);
523          }
524        }
525
526        // Remove hfiles not present in the snapshot
527        for (String hfileName : familyFiles) {
528          Path hfile = new Path(familyDir, hfileName);
529          if (!fs.getFileStatus(hfile).isDirectory()) {
530            LOG.trace("Removing HFile=" + hfileName + " not present in snapshot=" + snapshotName
531              + " from region=" + regionInfo.getEncodedName() + " table=" + tableName);
532            HFileArchiver.archiveStoreFile(conf, fs, regionInfo, tableDir, family, hfile);
533          }
534        }
535
536        // Restore Missing files
537        for (SnapshotRegionManifest.StoreFile storeFile : hfilesToAdd) {
538          LOG.debug("Restoring missing HFileLink " + storeFile.getName() + " of snapshot="
539            + snapshotName + " to region=" + regionInfo.getEncodedName() + " table=" + tableName);
540          String fileName =
541            restoreStoreFile(familyDir, regionInfo, storeFile, createBackRefs, tracker);
542          // mark the reference file to be added to tracker
543          filesToTrack.add(tracker.getStoreFileInfo(new Path(familyDir, fileName), true));
544        }
545      } else {
546        // Family doesn't exists in the snapshot
547        LOG.trace("Removing family=" + Bytes.toString(family) + " in snapshot=" + snapshotName
548          + " from region=" + regionInfo.getEncodedName() + " table=" + tableName);
549        HFileArchiver.archiveFamilyByFamilyDir(fs, conf, regionInfo, familyDir, family);
550        fs.delete(familyDir, true);
551      }
552
553      // simply reset list of tracked files with the matching files
554      // and the extra one present in the snapshot
555      tracker.set(filesToTrack);
556    }
557
558    // Add families not present in the table
559    for (Map.Entry<String, List<SnapshotRegionManifest.StoreFile>> familyEntry : snapshotFiles
560      .entrySet()) {
561      Path familyDir = new Path(regionDir, familyEntry.getKey());
562      StoreFileTracker tracker =
563        StoreFileTrackerFactory.create(conf, true, StoreContext.getBuilder()
564          .withFamilyStoreDirectoryPath(familyDir).withRegionFileSystem(regionFS).build());
565      List<StoreFileInfo> files = new ArrayList<>();
566      if (!fs.mkdirs(familyDir)) {
567        throw new IOException("Unable to create familyDir=" + familyDir);
568      }
569
570      for (SnapshotRegionManifest.StoreFile storeFile : familyEntry.getValue()) {
571        LOG.trace("Adding HFileLink (Not present in the table) " + storeFile.getName()
572          + " of snapshot " + snapshotName + " to table=" + tableName);
573        String fileName =
574          restoreStoreFile(familyDir, regionInfo, storeFile, createBackRefs, tracker);
575        files.add(tracker.getStoreFileInfo(new Path(familyDir, fileName), true));
576      }
577      tracker.set(files);
578    }
579  }
580
581  private Set<String> getTableRegionFamilyFiles(final Path familyDir) throws IOException {
582    FileStatus[] hfiles = CommonFSUtils.listStatus(fs, familyDir);
583    if (hfiles == null) {
584      return Collections.emptySet();
585    }
586
587    Set<String> familyFiles = new HashSet<>(hfiles.length);
588    for (int i = 0; i < hfiles.length; ++i) {
589      String hfileName = hfiles[i].getPath().getName();
590      familyFiles.add(hfileName);
591    }
592
593    return familyFiles;
594  }
595
596  /**
597   * Clone specified regions. For each region create a new region and create a HFileLink for each
598   * hfile.
599   */
600  private RegionInfo[] cloneHdfsRegions(final ThreadPoolExecutor exec,
601    final Map<String, SnapshotRegionManifest> regionManifests, final List<RegionInfo> regions)
602    throws IOException {
603    if (regions == null || regions.isEmpty()) return null;
604
605    final Map<String, RegionInfo> snapshotRegions = new HashMap<>(regions.size());
606    final String snapshotName = snapshotDesc.getName();
607
608    // clone region info (change embedded tableName with the new one)
609    RegionInfo[] clonedRegionsInfo = new RegionInfo[regions.size()];
610    for (int i = 0; i < clonedRegionsInfo.length; ++i) {
611      // clone the region info from the snapshot region info
612      RegionInfo snapshotRegionInfo = regions.get(i);
613      clonedRegionsInfo[i] = cloneRegionInfo(snapshotRegionInfo);
614
615      // add the region name mapping between snapshot and cloned
616      String snapshotRegionName = snapshotRegionInfo.getEncodedName();
617      String clonedRegionName = clonedRegionsInfo[i].getEncodedName();
618      regionsMap.put(Bytes.toBytes(snapshotRegionName), Bytes.toBytes(clonedRegionName));
619      LOG.info("clone region=" + snapshotRegionName + " as " + clonedRegionName + " in snapshot "
620        + snapshotName);
621
622      // Add mapping between cloned region name and snapshot region info
623      snapshotRegions.put(clonedRegionName, snapshotRegionInfo);
624    }
625
626    // create the regions on disk
627    ModifyRegionUtils.createRegions(exec, conf, rootDir, tableDesc, clonedRegionsInfo,
628      new ModifyRegionUtils.RegionFillTask() {
629        @Override
630        public void fillRegion(final HRegion region) throws IOException {
631          RegionInfo snapshotHri = snapshotRegions.get(region.getRegionInfo().getEncodedName());
632          cloneRegion(region, snapshotHri, regionManifests.get(snapshotHri.getEncodedName()));
633        }
634      });
635
636    return clonedRegionsInfo;
637  }
638
639  /**
640   * Clone the mob region. For the region create a new region and create a HFileLink for each hfile.
641   */
642  private void cloneHdfsMobRegion(final Map<String, SnapshotRegionManifest> regionManifests,
643    final RegionInfo region) throws IOException {
644    // clone region info (change embedded tableName with the new one)
645    Path clonedRegionPath = MobUtils.getMobRegionPath(rootDir, tableDesc.getTableName());
646    cloneRegion(MobUtils.getMobRegionInfo(tableDesc.getTableName()), clonedRegionPath, region,
647      regionManifests.get(region.getEncodedName()));
648  }
649
650  /**
651   * Clone region directory content from the snapshot info. Each region is encoded with the table
652   * name, so the cloned region will have a different region name. Instead of copying the hfiles a
653   * HFileLink is created.
654   * @param regionDir {@link Path} cloned dir
655   */
656  private void cloneRegion(final RegionInfo newRegionInfo, final Path regionDir,
657    final RegionInfo snapshotRegionInfo, final SnapshotRegionManifest manifest) throws IOException {
658    final String tableName = tableDesc.getTableName().getNameAsString();
659    final String snapshotName = snapshotDesc.getName();
660    for (SnapshotRegionManifest.FamilyFiles familyFiles : manifest.getFamilyFilesList()) {
661      Path familyDir = new Path(regionDir, familyFiles.getFamilyName().toStringUtf8());
662      List<StoreFileInfo> clonedFiles = new ArrayList<>();
663      Path regionPath = new Path(tableDir, newRegionInfo.getEncodedName());
664      HRegionFileSystem regionFS = (fs.exists(regionPath))
665        ? HRegionFileSystem.openRegionFromFileSystem(conf, fs, tableDir, newRegionInfo, false)
666        : HRegionFileSystem.createRegionOnFileSystem(conf, fs, tableDir, newRegionInfo);
667
668      Configuration sftConf = StoreUtils.createStoreConfiguration(conf, tableDesc,
669        tableDesc.getColumnFamily(familyFiles.getFamilyName().toByteArray()));
670      StoreFileTracker tracker =
671        StoreFileTrackerFactory.create(sftConf, true, StoreContext.getBuilder()
672          .withFamilyStoreDirectoryPath(familyDir).withRegionFileSystem(regionFS).build());
673      for (SnapshotRegionManifest.StoreFile storeFile : familyFiles.getStoreFilesList()) {
674        LOG.info("Adding HFileLink " + storeFile.getName() + " from cloned region " + "in snapshot "
675          + snapshotName + " to table=" + tableName);
676        if (MobUtils.isMobRegionInfo(newRegionInfo)) {
677          String mobFileName =
678            HFileLink.createHFileLinkName(snapshotRegionInfo, storeFile.getName());
679          Path mobPath = new Path(familyDir, mobFileName);
680          if (fs.exists(mobPath)) {
681            fs.delete(mobPath, true);
682          }
683          restoreStoreFile(familyDir, snapshotRegionInfo, storeFile, createBackRefs, tracker);
684        } else {
685          String file =
686            restoreStoreFile(familyDir, snapshotRegionInfo, storeFile, createBackRefs, tracker);
687          clonedFiles.add(tracker.getStoreFileInfo(new Path(familyDir, file), true));
688        }
689      }
690      // we don't need to track files under mobdir
691      if (!MobUtils.isMobRegionInfo(newRegionInfo)) {
692        tracker.set(clonedFiles);
693      }
694    }
695
696  }
697
698  /**
699   * Clone region directory content from the snapshot info. Each region is encoded with the table
700   * name, so the cloned region will have a different region name. Instead of copying the hfiles a
701   * HFileLink is created.
702   * @param region {@link HRegion} cloned
703   */
704  private void cloneRegion(final HRegion region, final RegionInfo snapshotRegionInfo,
705    final SnapshotRegionManifest manifest) throws IOException {
706    cloneRegion(region.getRegionInfo(), new Path(tableDir, region.getRegionInfo().getEncodedName()),
707      snapshotRegionInfo, manifest);
708  }
709
710  /**
711   * Create a new {@link HFileLink} to reference the store file.
712   * <p>
713   * The store file in the snapshot can be a simple hfile, an HFileLink or a reference.
714   * <ul>
715   * <li>hfile: abc -> table=region-abc
716   * <li>reference: abc.1234 -> table=region-abc.1234
717   * <li>hfilelink: table=region-hfile -> table=region-hfile
718   * </ul>
719   * @param familyDir     destination directory for the store file
720   * @param regionInfo    destination region info for the table
721   * @param createBackRef - Whether back reference should be created. Defaults to true.
722   * @param storeFile     store file name (can be a Reference, HFileLink or simple HFile)
723   */
724  private String restoreStoreFile(final Path familyDir, final RegionInfo regionInfo,
725    final SnapshotRegionManifest.StoreFile storeFile, final boolean createBackRef,
726    final StoreFileTracker tracker) throws IOException {
727    String hfileName = storeFile.getName();
728    if (HFileLink.isHFileLink(hfileName)) {
729      return HFileLink.createFromHFileLink(conf, fs, familyDir, hfileName, createBackRef);
730    } else if (StoreFileInfo.isReference(hfileName)) {
731      return restoreReferenceFile(familyDir, regionInfo, storeFile, tracker);
732    } else {
733      return HFileLink.create(conf, fs, familyDir, regionInfo, hfileName, createBackRef);
734    }
735  }
736
737  /**
738   * Create a new {@link Reference} as copy of the source one.
739   * <p>
740   * <blockquote>
741   *
742   * <pre>
743   * The source table looks like:
744   *    1234/abc      (original file)
745   *    5678/abc.1234 (reference file)
746   *
747   * After the clone operation looks like:
748   *   wxyz/table=1234-abc
749   *   stuv/table=1234-abc.wxyz
750   *
751   * NOTE that the region name in the clone changes (md5 of regioninfo)
752   * and the reference should reflect that change.
753   * </pre>
754   *
755   * </blockquote>
756   * @param familyDir  destination directory for the store file
757   * @param regionInfo destination region info for the table
758   * @param storeFile  reference file name
759   */
760  private String restoreReferenceFile(final Path familyDir, final RegionInfo regionInfo,
761    final SnapshotRegionManifest.StoreFile storeFile, final StoreFileTracker tracker)
762    throws IOException {
763    String hfileName = storeFile.getName();
764
765    // Extract the referred information (hfile name and parent region)
766    Path refPath =
767      StoreFileInfo
768        .getReferredToFile(
769          new Path(
770            new Path(
771              new Path(new Path(snapshotTable.getNamespaceAsString(),
772                snapshotTable.getQualifierAsString()), regionInfo.getEncodedName()),
773              familyDir.getName()),
774            hfileName));
775    String snapshotRegionName = refPath.getParent().getParent().getName();
776    String fileName = refPath.getName();
777
778    // The new reference should have the cloned region name as parent, if it is a clone.
779    String clonedRegionName = Bytes.toString(regionsMap.get(Bytes.toBytes(snapshotRegionName)));
780    if (clonedRegionName == null) clonedRegionName = snapshotRegionName;
781
782    // The output file should be a reference link table=snapshotRegion-fileName.clonedRegionName
783    Path linkPath = null;
784    String refLink = fileName;
785    if (!HFileLink.isHFileLink(fileName)) {
786      refLink = HFileLink.createHFileLinkName(snapshotTable, snapshotRegionName, fileName);
787      linkPath = new Path(familyDir,
788        HFileLink.createHFileLinkName(snapshotTable, regionInfo.getEncodedName(), hfileName));
789    }
790
791    Path outPath = new Path(familyDir, refLink + '.' + clonedRegionName);
792
793    // Create the new reference
794    if (storeFile.hasReference()) {
795      Reference reference = Reference.convert(storeFile.getReference());
796      tracker.createReference(reference, outPath);
797    } else {
798      InputStream in;
799      if (linkPath != null) {
800        in = HFileLink.buildFromHFileLinkPattern(conf, linkPath).open(fs);
801      } else {
802        linkPath = new Path(new Path(
803          HRegion.getRegionDir(snapshotManifest.getSnapshotDir(), regionInfo.getEncodedName()),
804          familyDir.getName()), hfileName);
805        in = fs.open(linkPath);
806      }
807      OutputStream out = fs.create(outPath);
808      IOUtils.copyBytes(in, out, conf);
809    }
810
811    // Add the daughter region to the map
812    String regionName = Bytes.toString(regionsMap.get(regionInfo.getEncodedNameAsBytes()));
813    if (regionName == null) {
814      regionName = regionInfo.getEncodedName();
815    }
816    LOG.debug("Restore reference " + regionName + " to " + clonedRegionName);
817    synchronized (parentsMap) {
818      Pair<String, String> daughters = parentsMap.get(clonedRegionName);
819      if (daughters == null) {
820        // In case one side of the split is already compacted, regionName is put as both first and
821        // second of Pair
822        daughters = new Pair<>(regionName, regionName);
823        parentsMap.put(clonedRegionName, daughters);
824      } else if (!regionName.equals(daughters.getFirst())) {
825        daughters.setSecond(regionName);
826      }
827    }
828    return outPath.getName();
829  }
830
831  /**
832   * Create a new {@link RegionInfo} from the snapshot region info. Keep the same startKey, endKey,
833   * regionId and split information but change the table name.
834   * @param snapshotRegionInfo Info for region to clone.
835   * @return the new HRegion instance
836   */
837  public RegionInfo cloneRegionInfo(final RegionInfo snapshotRegionInfo) {
838    return cloneRegionInfo(tableDesc.getTableName(), snapshotRegionInfo);
839  }
840
841  public static RegionInfo cloneRegionInfo(TableName tableName, RegionInfo snapshotRegionInfo) {
842    return RegionInfoBuilder.newBuilder(tableName).setStartKey(snapshotRegionInfo.getStartKey())
843      .setEndKey(snapshotRegionInfo.getEndKey()).setSplit(snapshotRegionInfo.isSplit())
844      .setRegionId(snapshotRegionInfo.getRegionId()).setOffline(snapshotRegionInfo.isOffline())
845      .build();
846  }
847
848  /** Returns the set of the regions contained in the table */
849  private List<RegionInfo> getTableRegions() throws IOException {
850    LOG.debug("get table regions: " + tableDir);
851    FileStatus[] regionDirs =
852      CommonFSUtils.listStatus(fs, tableDir, new FSUtils.RegionDirFilter(fs));
853    if (regionDirs == null) {
854      return null;
855    }
856
857    List<RegionInfo> regions = new ArrayList<>(regionDirs.length);
858    for (int i = 0; i < regionDirs.length; ++i) {
859      RegionInfo hri = HRegionFileSystem.loadRegionInfoFileContent(fs, regionDirs[i].getPath());
860      regions.add(hri);
861    }
862    LOG.debug("found " + regions.size() + " regions for table="
863      + tableDesc.getTableName().getNameAsString());
864    return regions;
865  }
866
867  /**
868   * Copy the snapshot files for a snapshot scanner, discards meta changes.
869   */
870  public static RestoreMetaChanges copySnapshotForScanner(Configuration conf, FileSystem fs,
871    Path rootDir, Path restoreDir, String snapshotName) throws IOException {
872    // ensure that restore dir is not under root dir
873    if (!restoreDir.getFileSystem(conf).getUri().equals(rootDir.getFileSystem(conf).getUri())) {
874      throw new IllegalArgumentException(
875        "Filesystems for restore directory and HBase root " + "directory should be the same");
876    }
877    if (restoreDir.toUri().getPath().startsWith(rootDir.toUri().getPath() + "/")) {
878      throw new IllegalArgumentException("Restore directory cannot be a sub directory of HBase "
879        + "root directory. RootDir: " + rootDir + ", restoreDir: " + restoreDir);
880    }
881
882    Path snapshotDir = SnapshotDescriptionUtils.getCompletedSnapshotDir(snapshotName, rootDir);
883    SnapshotDescription snapshotDesc = SnapshotDescriptionUtils.readSnapshotInfo(fs, snapshotDir);
884    // check if the snapshot is expired.
885    boolean isExpired = SnapshotDescriptionUtils.isExpiredSnapshot(snapshotDesc.getTtl(),
886      snapshotDesc.getCreationTime(), EnvironmentEdgeManager.currentTime());
887    if (isExpired) {
888      throw new SnapshotTTLExpiredException(ProtobufUtil.createSnapshotDesc(snapshotDesc));
889    }
890    SnapshotManifest manifest = SnapshotManifest.open(conf, fs, snapshotDir, snapshotDesc);
891
892    MonitoredTask status = TaskMonitor.get()
893      .createStatus("Restoring  snapshot '" + snapshotName + "' to directory " + restoreDir);
894    ForeignExceptionDispatcher monitor = new ForeignExceptionDispatcher();
895
896    // we send createBackRefs=false so that restored hfiles do not create back reference links
897    // in the base hbase root dir.
898    RestoreSnapshotHelper helper = new RestoreSnapshotHelper(conf, fs, manifest,
899      manifest.getTableDescriptor(), restoreDir, monitor, status, false);
900    RestoreMetaChanges metaChanges = helper.restoreHdfsRegions(); // TODO: parallelize.
901
902    if (LOG.isDebugEnabled()) {
903      LOG.debug("Restored table dir:" + restoreDir);
904      CommonFSUtils.logFileSystemState(fs, restoreDir, LOG);
905    }
906    return metaChanges;
907  }
908
909  public static void restoreSnapshotAcl(SnapshotDescription snapshot, TableName newTableName,
910    Configuration conf) throws IOException {
911    if (snapshot.hasUsersAndPermissions() && snapshot.getUsersAndPermissions() != null) {
912      LOG.info("Restore snapshot acl to table. snapshot: " + snapshot + ", table: " + newTableName);
913      ListMultimap<String, Permission> perms =
914        ShadedAccessControlUtil.toUserTablePermissions(snapshot.getUsersAndPermissions());
915      try (Connection conn = ConnectionFactory.createConnection(conf)) {
916        for (Entry<String, Permission> e : perms.entries()) {
917          String user = e.getKey();
918          TablePermission tablePerm = (TablePermission) e.getValue();
919          AccessControlClient.grant(conn, newTableName, user, tablePerm.getFamily(),
920            tablePerm.getQualifier(), tablePerm.getActions());
921        }
922      } catch (Throwable e) {
923        throw new IOException("Grant acl into newly creatd table failed. snapshot: " + snapshot
924          + ", table: " + newTableName, e);
925      }
926    }
927  }
928}