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