View Javadoc

1   /**
2    * Licensed to the Apache Software Foundation (ASF) under one
3    * or more contributor license agreements.  See the NOTICE file
4    * distributed with this work for additional information
5    * regarding copyright ownership.  The ASF licenses this file
6    * to you under the Apache License, Version 2.0 (the
7    * "License"); you may not use this file except in compliance
8    * with the License.  You may obtain a copy of the License at
9    *
10   *     http://www.apache.org/licenses/LICENSE-2.0
11   *
12   * Unless required by applicable law or agreed to in writing, software
13   * distributed under the License is distributed on an "AS IS" BASIS,
14   * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15   * See the License for the specific language governing permissions and
16   * limitations under the License.
17   */
18  
19  package org.apache.hadoop.hbase.snapshot;
20  
21  import java.io.IOException;
22  import java.io.InputStream;
23  import java.io.OutputStream;
24  import java.util.ArrayList;
25  import java.util.Arrays;
26  import java.util.HashMap;
27  import java.util.HashSet;
28  import java.util.LinkedList;
29  import java.util.List;
30  import java.util.Map;
31  import java.util.Set;
32  import java.util.TreeMap;
33  import java.util.concurrent.ThreadPoolExecutor;
34  
35  import org.apache.commons.logging.Log;
36  import org.apache.commons.logging.LogFactory;
37  import org.apache.hadoop.hbase.classification.InterfaceAudience;
38  import org.apache.hadoop.conf.Configuration;
39  import org.apache.hadoop.fs.FileStatus;
40  import org.apache.hadoop.fs.FileSystem;
41  import org.apache.hadoop.fs.Path;
42  import org.apache.hadoop.hbase.HColumnDescriptor;
43  import org.apache.hadoop.hbase.HRegionInfo;
44  import org.apache.hadoop.hbase.HTableDescriptor;
45  import org.apache.hadoop.hbase.TableName;
46  import org.apache.hadoop.hbase.backup.HFileArchiver;
47  import org.apache.hadoop.hbase.MetaTableAccessor;
48  import org.apache.hadoop.hbase.client.Connection;
49  import org.apache.hadoop.hbase.errorhandling.ForeignExceptionDispatcher;
50  import org.apache.hadoop.hbase.io.HFileLink;
51  import org.apache.hadoop.hbase.io.Reference;
52  import org.apache.hadoop.hbase.io.ImmutableBytesWritable;
53  import org.apache.hadoop.hbase.monitoring.MonitoredTask;
54  import org.apache.hadoop.hbase.monitoring.TaskMonitor;
55  import org.apache.hadoop.hbase.protobuf.generated.HBaseProtos.SnapshotDescription;
56  import org.apache.hadoop.hbase.protobuf.generated.SnapshotProtos.SnapshotRegionManifest;
57  import org.apache.hadoop.hbase.regionserver.HRegion;
58  import org.apache.hadoop.hbase.regionserver.HRegionFileSystem;
59  import org.apache.hadoop.hbase.regionserver.StoreFileInfo;
60  import org.apache.hadoop.hbase.util.Bytes;
61  import org.apache.hadoop.hbase.util.FSUtils;
62  import org.apache.hadoop.hbase.util.ModifyRegionUtils;
63  import org.apache.hadoop.hbase.util.Pair;
64  import org.apache.hadoop.io.IOUtils;
65  
66  /**
67   * Helper to Restore/Clone a Snapshot
68   *
69   * <p>The helper assumes that a table is already created, and by calling restore()
70   * the content present in the snapshot will be restored as the new content of the table.
71   *
72   * <p>Clone from Snapshot: If the target table is empty, the restore operation
73   * is just a "clone operation", where the only operations are:
74   * <ul>
75   *  <li>for each region in the snapshot create a new region
76   *    (note that the region will have a different name, since the encoding contains the table name)
77   *  <li>for each file in the region create a new HFileLink to point to the original file.
78   *  <li>restore the logs, if any
79   * </ul>
80   *
81   * <p>Restore from Snapshot:
82   * <ul>
83   *  <li>for each region in the table verify which are available in the snapshot and which are not
84   *    <ul>
85   *    <li>if the region is not present in the snapshot, remove it.
86   *    <li>if the region is present in the snapshot
87   *      <ul>
88   *      <li>for each file in the table region verify which are available in the snapshot
89   *        <ul>
90   *          <li>if the hfile is not present in the snapshot, remove it
91   *          <li>if the hfile is present, keep it (nothing to do)
92   *        </ul>
93   *      <li>for each file in the snapshot region but not in the table
94   *        <ul>
95   *          <li>create a new HFileLink that point to the original file
96   *        </ul>
97   *      </ul>
98   *    </ul>
99   *  <li>for each region in the snapshot not present in the current table state
100  *    <ul>
101  *    <li>create a new region and for each file in the region create a new HFileLink
102  *      (This is the same as the clone operation)
103  *    </ul>
104  *  <li>restore the logs, if any
105  * </ul>
106  */
107 @InterfaceAudience.Private
108 public class RestoreSnapshotHelper {
109   private static final Log LOG = LogFactory.getLog(RestoreSnapshotHelper.class);
110 
111   private final Map<byte[], byte[]> regionsMap =
112         new TreeMap<byte[], byte[]>(Bytes.BYTES_COMPARATOR);
113 
114   private final Map<String, Pair<String, String> > parentsMap =
115       new HashMap<String, Pair<String, String> >();
116 
117   private final ForeignExceptionDispatcher monitor;
118   private final MonitoredTask status;
119 
120   private final SnapshotManifest snapshotManifest;
121   private final SnapshotDescription snapshotDesc;
122   private final TableName snapshotTable;
123 
124   private final HTableDescriptor tableDesc;
125   private final Path rootDir;
126   private final Path tableDir;
127 
128   private final Configuration conf;
129   private final FileSystem fs;
130   private final boolean createBackRefs;
131 
132   public RestoreSnapshotHelper(final Configuration conf,
133       final FileSystem fs,
134       final SnapshotManifest manifest,
135       final HTableDescriptor tableDescriptor,
136       final Path rootDir,
137       final ForeignExceptionDispatcher monitor,
138       final MonitoredTask status) {
139     this(conf, fs, manifest, tableDescriptor, rootDir, monitor, status, true);
140   }
141 
142   public RestoreSnapshotHelper(final Configuration conf,
143       final FileSystem fs,
144       final SnapshotManifest manifest,
145       final HTableDescriptor tableDescriptor,
146       final Path rootDir,
147       final ForeignExceptionDispatcher monitor,
148       final MonitoredTask status,
149       final boolean createBackRefs)
150   {
151     this.fs = fs;
152     this.conf = conf;
153     this.snapshotManifest = manifest;
154     this.snapshotDesc = manifest.getSnapshotDescription();
155     this.snapshotTable = TableName.valueOf(snapshotDesc.getTable());
156     this.tableDesc = tableDescriptor;
157     this.rootDir = rootDir;
158     this.tableDir = FSUtils.getTableDir(rootDir, tableDesc.getTableName());
159     this.monitor = monitor;
160     this.status = status;
161     this.createBackRefs = createBackRefs;
162   }
163 
164   /**
165    * Restore the on-disk table to a specified snapshot state.
166    * @return the set of regions touched by the restore operation
167    */
168   public RestoreMetaChanges restoreHdfsRegions() throws IOException {
169     ThreadPoolExecutor exec = SnapshotManifest.createExecutor(conf, "RestoreSnapshot");
170     try {
171       return restoreHdfsRegions(exec);
172     } finally {
173       exec.shutdown();
174     }
175   }
176 
177   private RestoreMetaChanges restoreHdfsRegions(final ThreadPoolExecutor exec) throws IOException {
178     LOG.debug("starting restore");
179 
180     Map<String, SnapshotRegionManifest> regionManifests = snapshotManifest.getRegionManifestsMap();
181     if (regionManifests == null) {
182       LOG.warn("Nothing to restore. Snapshot " + snapshotDesc + " looks empty");
183       return null;
184     }
185 
186     RestoreMetaChanges metaChanges = new RestoreMetaChanges(parentsMap);
187 
188     // Take a copy of the manifest.keySet() since we are going to modify
189     // this instance, by removing the regions already present in the restore dir.
190     Set<String> regionNames = new HashSet<String>(regionManifests.keySet());
191 
192     // Identify which region are still available and which not.
193     // NOTE: we rely upon the region name as: "table name, start key, end key"
194     List<HRegionInfo> tableRegions = getTableRegions();
195     if (tableRegions != null) {
196       monitor.rethrowException();
197       for (HRegionInfo regionInfo: tableRegions) {
198         String regionName = regionInfo.getEncodedName();
199         if (regionNames.contains(regionName)) {
200           LOG.info("region to restore: " + regionName);
201           regionNames.remove(regionName);
202           metaChanges.addRegionToRestore(regionInfo);
203         } else {
204           LOG.info("region to remove: " + regionName);
205           metaChanges.addRegionToRemove(regionInfo);
206         }
207       }
208 
209       // Restore regions using the snapshot data
210       monitor.rethrowException();
211       status.setStatus("Restoring table regions...");
212       restoreHdfsRegions(exec, regionManifests, metaChanges.getRegionsToRestore());
213       status.setStatus("Finished restoring all table regions.");
214 
215       // Remove regions from the current table
216       monitor.rethrowException();
217       status.setStatus("Starting to delete excess regions from table");
218       removeHdfsRegions(exec, metaChanges.getRegionsToRemove());
219       status.setStatus("Finished deleting excess regions from table.");
220     }
221 
222     // Regions to Add: present in the snapshot but not in the current table
223     if (regionNames.size() > 0) {
224       List<HRegionInfo> regionsToAdd = new ArrayList<HRegionInfo>(regionNames.size());
225 
226       monitor.rethrowException();
227       for (String regionName: regionNames) {
228         LOG.info("region to add: " + regionName);
229         regionsToAdd.add(HRegionInfo.convert(regionManifests.get(regionName).getRegionInfo()));
230       }
231 
232       // Create new regions cloning from the snapshot
233       monitor.rethrowException();
234       status.setStatus("Cloning regions...");
235       HRegionInfo[] clonedRegions = cloneHdfsRegions(exec, regionManifests, regionsToAdd);
236       metaChanges.setNewRegions(clonedRegions);
237       status.setStatus("Finished cloning regions.");
238     }
239 
240     return metaChanges;
241   }
242 
243   /**
244    * Describe the set of operations needed to update hbase:meta after restore.
245    */
246   public static class RestoreMetaChanges {
247     private final Map<String, Pair<String, String> > parentsMap;
248 
249     private List<HRegionInfo> regionsToRestore = null;
250     private List<HRegionInfo> regionsToRemove = null;
251     private List<HRegionInfo> regionsToAdd = null;
252 
253     RestoreMetaChanges(final Map<String, Pair<String, String> > parentsMap) {
254       this.parentsMap = parentsMap;
255     }
256 
257     /**
258      * @return true if there're new regions
259      */
260     public boolean hasRegionsToAdd() {
261       return this.regionsToAdd != null && this.regionsToAdd.size() > 0;
262     }
263 
264     /**
265      * Returns the list of new regions added during the on-disk restore.
266      * The caller is responsible to add the regions to META.
267      * e.g MetaTableAccessor.addRegionsToMeta(...)
268      * @return the list of regions to add to META
269      */
270     public List<HRegionInfo> getRegionsToAdd() {
271       return this.regionsToAdd;
272     }
273 
274     /**
275      * @return true if there're regions to restore
276      */
277     public boolean hasRegionsToRestore() {
278       return this.regionsToRestore != null && this.regionsToRestore.size() > 0;
279     }
280 
281     /**
282      * Returns the list of 'restored regions' during the on-disk restore.
283      * The caller is responsible to add the regions to hbase:meta if not present.
284      * @return the list of regions restored
285      */
286     public List<HRegionInfo> getRegionsToRestore() {
287       return this.regionsToRestore;
288     }
289 
290     /**
291      * @return true if there're regions to remove
292      */
293     public boolean hasRegionsToRemove() {
294       return this.regionsToRemove != null && this.regionsToRemove.size() > 0;
295     }
296 
297     /**
298      * Returns the list of regions removed during the on-disk restore.
299      * The caller is responsible to remove the regions from META.
300      * e.g. MetaTableAccessor.deleteRegions(...)
301      * @return the list of regions to remove from META
302      */
303     public List<HRegionInfo> getRegionsToRemove() {
304       return this.regionsToRemove;
305     }
306 
307     void setNewRegions(final HRegionInfo[] hris) {
308       if (hris != null) {
309         regionsToAdd = Arrays.asList(hris);
310       } else {
311         regionsToAdd = null;
312       }
313     }
314 
315     void addRegionToRemove(final HRegionInfo hri) {
316       if (regionsToRemove == null) {
317         regionsToRemove = new LinkedList<HRegionInfo>();
318       }
319       regionsToRemove.add(hri);
320     }
321 
322     void addRegionToRestore(final HRegionInfo hri) {
323       if (regionsToRestore == null) {
324         regionsToRestore = new LinkedList<HRegionInfo>();
325       }
326       regionsToRestore.add(hri);
327     }
328 
329     public void updateMetaParentRegions(Connection connection,
330         final List<HRegionInfo> regionInfos) throws IOException {
331       if (regionInfos == null || parentsMap.isEmpty()) return;
332 
333       // Extract region names and offlined regions
334       Map<String, HRegionInfo> regionsByName = new HashMap<String, HRegionInfo>(regionInfos.size());
335       List<HRegionInfo> parentRegions = new LinkedList<>();
336       for (HRegionInfo regionInfo: regionInfos) {
337         if (regionInfo.isSplitParent()) {
338           parentRegions.add(regionInfo);
339         } else {
340           regionsByName.put(regionInfo.getEncodedName(), regionInfo);
341         }
342       }
343 
344       // Update Offline parents
345       for (HRegionInfo regionInfo: parentRegions) {
346         Pair<String, String> daughters = parentsMap.get(regionInfo.getEncodedName());
347         if (daughters == null) {
348           // The snapshot contains an unreferenced region.
349           // It will be removed by the CatalogJanitor.
350           LOG.warn("Skip update of unreferenced offline parent: " + regionInfo);
351           continue;
352         }
353 
354         // One side of the split is already compacted
355         if (daughters.getSecond() == null) {
356           daughters.setSecond(daughters.getFirst());
357         }
358 
359         LOG.debug("Update splits parent " + regionInfo.getEncodedName() + " -> " + daughters);
360         MetaTableAccessor.addRegionToMeta(connection, regionInfo,
361           regionsByName.get(daughters.getFirst()),
362           regionsByName.get(daughters.getSecond()));
363       }
364     }
365   }
366 
367   /**
368    * Remove specified regions from the file-system, using the archiver.
369    */
370   private void removeHdfsRegions(final ThreadPoolExecutor exec, final List<HRegionInfo> regions)
371       throws IOException {
372     if (regions == null || regions.size() == 0) return;
373     ModifyRegionUtils.editRegions(exec, regions, new ModifyRegionUtils.RegionEditTask() {
374       @Override
375       public void editRegion(final HRegionInfo hri) throws IOException {
376         HFileArchiver.archiveRegion(conf, fs, hri);
377       }
378     });
379   }
380 
381   /**
382    * Restore specified regions by restoring content to the snapshot state.
383    */
384   private void restoreHdfsRegions(final ThreadPoolExecutor exec,
385       final Map<String, SnapshotRegionManifest> regionManifests,
386       final List<HRegionInfo> regions) throws IOException {
387     if (regions == null || regions.size() == 0) return;
388     ModifyRegionUtils.editRegions(exec, regions, new ModifyRegionUtils.RegionEditTask() {
389       @Override
390       public void editRegion(final HRegionInfo hri) throws IOException {
391         restoreRegion(hri, regionManifests.get(hri.getEncodedName()));
392       }
393     });
394   }
395 
396   private Map<String, List<SnapshotRegionManifest.StoreFile>> getRegionHFileReferences(
397       final SnapshotRegionManifest manifest) {
398     Map<String, List<SnapshotRegionManifest.StoreFile>> familyMap =
399       new HashMap<String, List<SnapshotRegionManifest.StoreFile>>(manifest.getFamilyFilesCount());
400     for (SnapshotRegionManifest.FamilyFiles familyFiles: manifest.getFamilyFilesList()) {
401       familyMap.put(familyFiles.getFamilyName().toStringUtf8(),
402         new ArrayList<SnapshotRegionManifest.StoreFile>(familyFiles.getStoreFilesList()));
403     }
404     return familyMap;
405   }
406 
407   /**
408    * Restore region by removing files not in the snapshot
409    * and adding the missing ones from the snapshot.
410    */
411   private void restoreRegion(final HRegionInfo regionInfo,
412       final SnapshotRegionManifest regionManifest) throws IOException {
413     Map<String, List<SnapshotRegionManifest.StoreFile>> snapshotFiles =
414                 getRegionHFileReferences(regionManifest);
415 
416     Path regionDir = new Path(tableDir, regionInfo.getEncodedName());
417     String tableName = tableDesc.getTableName().getNameAsString();
418 
419     // Restore families present in the table
420     for (Path familyDir: FSUtils.getFamilyDirs(fs, regionDir)) {
421       byte[] family = Bytes.toBytes(familyDir.getName());
422       Set<String> familyFiles = getTableRegionFamilyFiles(familyDir);
423       List<SnapshotRegionManifest.StoreFile> snapshotFamilyFiles =
424           snapshotFiles.remove(familyDir.getName());
425       if (snapshotFamilyFiles != null) {
426         List<SnapshotRegionManifest.StoreFile> hfilesToAdd =
427             new ArrayList<SnapshotRegionManifest.StoreFile>();
428         for (SnapshotRegionManifest.StoreFile storeFile: snapshotFamilyFiles) {
429           if (familyFiles.contains(storeFile.getName())) {
430             // HFile already present
431             familyFiles.remove(storeFile.getName());
432           } else {
433             // HFile missing
434             hfilesToAdd.add(storeFile);
435           }
436         }
437 
438         // Remove hfiles not present in the snapshot
439         for (String hfileName: familyFiles) {
440           Path hfile = new Path(familyDir, hfileName);
441           LOG.trace("Removing hfile=" + hfileName +
442             " from region=" + regionInfo.getEncodedName() + " table=" + tableName);
443           HFileArchiver.archiveStoreFile(conf, fs, regionInfo, tableDir, family, hfile);
444         }
445 
446         // Restore Missing files
447         for (SnapshotRegionManifest.StoreFile storeFile: hfilesToAdd) {
448           LOG.debug("Adding HFileLink " + storeFile.getName() +
449             " to region=" + regionInfo.getEncodedName() + " table=" + tableName);
450           restoreStoreFile(familyDir, regionInfo, storeFile, createBackRefs);
451         }
452       } else {
453         // Family doesn't exists in the snapshot
454         LOG.trace("Removing family=" + Bytes.toString(family) +
455           " from region=" + regionInfo.getEncodedName() + " table=" + tableName);
456         HFileArchiver.archiveFamily(fs, conf, regionInfo, tableDir, family);
457         fs.delete(familyDir, true);
458       }
459     }
460 
461     // Add families not present in the table
462     for (Map.Entry<String, List<SnapshotRegionManifest.StoreFile>> familyEntry:
463                                                                       snapshotFiles.entrySet()) {
464       Path familyDir = new Path(regionDir, familyEntry.getKey());
465       if (!fs.mkdirs(familyDir)) {
466         throw new IOException("Unable to create familyDir=" + familyDir);
467       }
468 
469       for (SnapshotRegionManifest.StoreFile storeFile: familyEntry.getValue()) {
470         LOG.trace("Adding HFileLink " + storeFile.getName() + " to table=" + tableName);
471         restoreStoreFile(familyDir, regionInfo, storeFile, createBackRefs);
472       }
473     }
474   }
475 
476   /**
477    * @return The set of files in the specified family directory.
478    */
479   private Set<String> getTableRegionFamilyFiles(final Path familyDir) throws IOException {
480     Set<String> familyFiles = new HashSet<String>();
481 
482     FileStatus[] hfiles = FSUtils.listStatus(fs, familyDir);
483     if (hfiles == null) return familyFiles;
484 
485     for (FileStatus hfileRef: hfiles) {
486       String hfileName = hfileRef.getPath().getName();
487       familyFiles.add(hfileName);
488     }
489 
490     return familyFiles;
491   }
492 
493   /**
494    * Clone specified regions. For each region create a new region
495    * and create a HFileLink for each hfile.
496    */
497   private HRegionInfo[] cloneHdfsRegions(final ThreadPoolExecutor exec,
498       final Map<String, SnapshotRegionManifest> regionManifests,
499       final List<HRegionInfo> regions) throws IOException {
500     if (regions == null || regions.size() == 0) return null;
501 
502     final Map<String, HRegionInfo> snapshotRegions =
503       new HashMap<String, HRegionInfo>(regions.size());
504 
505     // clone region info (change embedded tableName with the new one)
506     HRegionInfo[] clonedRegionsInfo = new HRegionInfo[regions.size()];
507     for (int i = 0; i < clonedRegionsInfo.length; ++i) {
508       // clone the region info from the snapshot region info
509       HRegionInfo snapshotRegionInfo = regions.get(i);
510       clonedRegionsInfo[i] = cloneRegionInfo(snapshotRegionInfo);
511 
512       // add the region name mapping between snapshot and cloned
513       String snapshotRegionName = snapshotRegionInfo.getEncodedName();
514       String clonedRegionName = clonedRegionsInfo[i].getEncodedName();
515       regionsMap.put(Bytes.toBytes(snapshotRegionName), Bytes.toBytes(clonedRegionName));
516       LOG.info("clone region=" + snapshotRegionName + " as " + clonedRegionName);
517 
518       // Add mapping between cloned region name and snapshot region info
519       snapshotRegions.put(clonedRegionName, snapshotRegionInfo);
520     }
521 
522     // create the regions on disk
523     ModifyRegionUtils.createRegions(exec, conf, rootDir, tableDir,
524       tableDesc, clonedRegionsInfo, new ModifyRegionUtils.RegionFillTask() {
525         @Override
526         public void fillRegion(final HRegion region) throws IOException {
527           HRegionInfo snapshotHri = snapshotRegions.get(region.getRegionInfo().getEncodedName());
528           cloneRegion(region, snapshotHri, regionManifests.get(snapshotHri.getEncodedName()));
529         }
530       });
531 
532     return clonedRegionsInfo;
533   }
534 
535   /**
536    * Clone region directory content from the snapshot info.
537    *
538    * Each region is encoded with the table name, so the cloned region will have
539    * a different region name.
540    *
541    * Instead of copying the hfiles a HFileLink is created.
542    *
543    * @param region {@link HRegion} cloned
544    * @param snapshotRegionInfo
545    */
546   private void cloneRegion(final HRegion region, final HRegionInfo snapshotRegionInfo,
547       final SnapshotRegionManifest manifest) throws IOException {
548     final Path regionDir = new Path(tableDir, region.getRegionInfo().getEncodedName());
549     final String tableName = tableDesc.getTableName().getNameAsString();
550     for (SnapshotRegionManifest.FamilyFiles familyFiles: manifest.getFamilyFilesList()) {
551       Path familyDir = new Path(regionDir, familyFiles.getFamilyName().toStringUtf8());
552       for (SnapshotRegionManifest.StoreFile storeFile: familyFiles.getStoreFilesList()) {
553         LOG.info("Adding HFileLink " + storeFile.getName() + " to table=" + tableName);
554         restoreStoreFile(familyDir, snapshotRegionInfo, storeFile, createBackRefs);
555       }
556     }
557   }
558 
559   /**
560    * Create a new {@link HFileLink} to reference the store file.
561    * <p>The store file in the snapshot can be a simple hfile, an HFileLink or a reference.
562    * <ul>
563    *   <li>hfile: abc -> table=region-abc
564    *   <li>reference: abc.1234 -> table=region-abc.1234
565    *   <li>hfilelink: table=region-hfile -> table=region-hfile
566    * </ul>
567    * @param familyDir destination directory for the store file
568    * @param regionInfo destination region info for the table
569    * @param storeFile store file name (can be a Reference, HFileLink or simple HFile)
570    * @param createBackRef - Whether back reference should be created. Defaults to true.
571    */
572   private void restoreStoreFile(final Path familyDir, final HRegionInfo regionInfo,
573       final SnapshotRegionManifest.StoreFile storeFile, final boolean createBackRef)
574           throws IOException {
575     String hfileName = storeFile.getName();
576     if (HFileLink.isHFileLink(hfileName)) {
577       HFileLink.createFromHFileLink(conf, fs, familyDir, hfileName, createBackRef);
578     } else if (StoreFileInfo.isReference(hfileName)) {
579       restoreReferenceFile(familyDir, regionInfo, storeFile);
580     } else {
581       HFileLink.create(conf, fs, familyDir, regionInfo, hfileName, createBackRef);
582     }
583   }
584 
585   /**
586    * Create a new {@link Reference} as copy of the source one.
587    * <p><blockquote><pre>
588    * The source table looks like:
589    *    1234/abc      (original file)
590    *    5678/abc.1234 (reference file)
591    *
592    * After the clone operation looks like:
593    *   wxyz/table=1234-abc
594    *   stuv/table=1234-abc.wxyz
595    *
596    * NOTE that the region name in the clone changes (md5 of regioninfo)
597    * and the reference should reflect that change.
598    * </pre></blockquote>
599    * @param familyDir destination directory for the store file
600    * @param regionInfo destination region info for the table
601    * @param hfileName reference file name
602    */
603   private void restoreReferenceFile(final Path familyDir, final HRegionInfo regionInfo,
604       final SnapshotRegionManifest.StoreFile storeFile) throws IOException {
605     String hfileName = storeFile.getName();
606 
607     // Extract the referred information (hfile name and parent region)
608     Path refPath =
609         StoreFileInfo.getReferredToFile(new Path(new Path(new Path(new Path(snapshotTable
610             .getNamespaceAsString(), snapshotTable.getQualifierAsString()), regionInfo
611             .getEncodedName()), familyDir.getName()), hfileName)); 
612     String snapshotRegionName = refPath.getParent().getParent().getName();
613     String fileName = refPath.getName();
614 
615     // The new reference should have the cloned region name as parent, if it is a clone.
616     String clonedRegionName = Bytes.toString(regionsMap.get(Bytes.toBytes(snapshotRegionName)));
617     if (clonedRegionName == null) clonedRegionName = snapshotRegionName;
618 
619     // The output file should be a reference link table=snapshotRegion-fileName.clonedRegionName
620     Path linkPath = null;
621     String refLink = fileName;
622     if (!HFileLink.isHFileLink(fileName)) {
623       refLink = HFileLink.createHFileLinkName(snapshotTable, snapshotRegionName, fileName);
624       linkPath = new Path(familyDir,
625         HFileLink.createHFileLinkName(snapshotTable, regionInfo.getEncodedName(), hfileName));
626     }
627 
628     Path outPath = new Path(familyDir, refLink + '.' + clonedRegionName);
629 
630     // Create the new reference
631     if (storeFile.hasReference()) {
632       Reference reference = Reference.convert(storeFile.getReference());
633       reference.write(fs, outPath);
634     } else {
635       InputStream in;
636       if (linkPath != null) {
637         in = HFileLink.buildFromHFileLinkPattern(conf, linkPath).open(fs);
638       } else {
639         linkPath = new Path(new Path(HRegion.getRegionDir(snapshotManifest.getSnapshotDir(),
640                         regionInfo.getEncodedName()), familyDir.getName()), hfileName);
641         in = fs.open(linkPath);
642       }
643       OutputStream out = fs.create(outPath);
644       IOUtils.copyBytes(in, out, conf);
645     }
646 
647     // Add the daughter region to the map
648     String regionName = Bytes.toString(regionsMap.get(regionInfo.getEncodedNameAsBytes()));
649     LOG.debug("Restore reference " + regionName + " to " + clonedRegionName);
650     synchronized (parentsMap) {
651       Pair<String, String> daughters = parentsMap.get(clonedRegionName);
652       if (daughters == null) {
653         daughters = new Pair<String, String>(regionName, null);
654         parentsMap.put(clonedRegionName, daughters);
655       } else if (!regionName.equals(daughters.getFirst())) {
656         daughters.setSecond(regionName);
657       }
658     }
659   }
660 
661   /**
662    * Create a new {@link HRegionInfo} from the snapshot region info.
663    * Keep the same startKey, endKey, regionId and split information but change
664    * the table name.
665    *
666    * @param snapshotRegionInfo Info for region to clone.
667    * @return the new HRegion instance
668    */
669   public HRegionInfo cloneRegionInfo(final HRegionInfo snapshotRegionInfo) {
670     HRegionInfo regionInfo = new HRegionInfo(tableDesc.getTableName(),
671                       snapshotRegionInfo.getStartKey(), snapshotRegionInfo.getEndKey(),
672                       snapshotRegionInfo.isSplit(), snapshotRegionInfo.getRegionId());
673     regionInfo.setOffline(snapshotRegionInfo.isOffline());
674     return regionInfo;
675   }
676 
677   /**
678    * @return the set of the regions contained in the table
679    */
680   private List<HRegionInfo> getTableRegions() throws IOException {
681     LOG.debug("get table regions: " + tableDir);
682     FileStatus[] regionDirs = FSUtils.listStatus(fs, tableDir, new FSUtils.RegionDirFilter(fs));
683     if (regionDirs == null) return null;
684 
685     List<HRegionInfo> regions = new LinkedList<HRegionInfo>();
686     for (FileStatus regionDir: regionDirs) {
687       HRegionInfo hri = HRegionFileSystem.loadRegionInfoFileContent(fs, regionDir.getPath());
688       regions.add(hri);
689     }
690     LOG.debug("found " + regions.size() + " regions for table=" +
691         tableDesc.getTableName().getNameAsString());
692     return regions;
693   }
694 
695   /**
696    * Create a new table descriptor cloning the snapshot table schema.
697    *
698    * @param snapshotTableDescriptor
699    * @param tableName
700    * @return cloned table descriptor
701    * @throws IOException
702    */
703   public static HTableDescriptor cloneTableSchema(final HTableDescriptor snapshotTableDescriptor,
704       final TableName tableName) throws IOException {
705     HTableDescriptor htd = new HTableDescriptor(tableName);
706     for (HColumnDescriptor hcd: snapshotTableDescriptor.getColumnFamilies()) {
707       htd.addFamily(hcd);
708     }
709     for (Map.Entry<ImmutableBytesWritable, ImmutableBytesWritable> e:
710         snapshotTableDescriptor.getValues().entrySet()) {
711       htd.setValue(e.getKey(), e.getValue());
712     }
713     for (Map.Entry<String, String> e: snapshotTableDescriptor.getConfiguration().entrySet()) {
714       htd.setConfiguration(e.getKey(), e.getValue());
715     }
716     return htd;
717   }
718 
719   /**
720    * Copy the snapshot files for a snapshot scanner, discards meta changes.
721    * @param conf
722    * @param fs
723    * @param rootDir
724    * @param restoreDir
725    * @param snapshotName
726    * @throws IOException
727    */
728   public static void copySnapshotForScanner(Configuration conf, FileSystem fs, Path rootDir,
729       Path restoreDir, String snapshotName) throws IOException {
730     // ensure that restore dir is not under root dir
731     if (!restoreDir.getFileSystem(conf).getUri().equals(rootDir.getFileSystem(conf).getUri())) {
732       throw new IllegalArgumentException("Filesystems for restore directory and HBase root directory " +
733           "should be the same");
734     }
735     if (restoreDir.toUri().getPath().startsWith(rootDir.toUri().getPath())) {
736       throw new IllegalArgumentException("Restore directory cannot be a sub directory of HBase " +
737           "root directory. RootDir: " + rootDir + ", restoreDir: " + restoreDir);
738     }
739 
740     Path snapshotDir = SnapshotDescriptionUtils.getCompletedSnapshotDir(snapshotName, rootDir);
741     SnapshotDescription snapshotDesc = SnapshotDescriptionUtils.readSnapshotInfo(fs, snapshotDir);
742     SnapshotManifest manifest = SnapshotManifest.open(conf, fs, snapshotDir, snapshotDesc);
743 
744     MonitoredTask status = TaskMonitor.get().createStatus(
745         "Restoring  snapshot '" + snapshotName + "' to directory " + restoreDir);
746     ForeignExceptionDispatcher monitor = new ForeignExceptionDispatcher();
747 
748     // we send createBackRefs=false so that restored hfiles do not create back reference links
749     // in the base hbase root dir.
750     RestoreSnapshotHelper helper = new RestoreSnapshotHelper(conf, fs,
751       manifest, manifest.getTableDescriptor(), restoreDir, monitor, status, false);
752     helper.restoreHdfsRegions(); // TODO: parallelize.
753 
754     if (LOG.isDebugEnabled()) {
755       LOG.debug("Restored table dir:" + restoreDir);
756       FSUtils.logFileSystemState(fs, restoreDir, LOG);
757     }
758   }
759 }