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.FileNotFoundException;
22  import java.io.IOException;
23  import java.io.InterruptedIOException;
24  import java.util.HashSet;
25  import java.util.List;
26  import java.util.Set;
27  import java.util.concurrent.Callable;
28  import java.util.concurrent.ExecutorService;
29  import java.util.concurrent.ExecutionException;
30  import java.util.concurrent.ExecutorCompletionService;
31  
32  import org.apache.commons.logging.Log;
33  import org.apache.commons.logging.LogFactory;
34  import org.apache.hadoop.hbase.classification.InterfaceAudience;
35  import org.apache.hadoop.conf.Configuration;
36  import org.apache.hadoop.fs.FileStatus;
37  import org.apache.hadoop.fs.FileSystem;
38  import org.apache.hadoop.fs.Path;
39  import org.apache.hadoop.hbase.HRegionInfo;
40  import org.apache.hadoop.hbase.TableName;
41  import org.apache.hadoop.hbase.io.HFileLink;
42  import org.apache.hadoop.hbase.protobuf.generated.HBaseProtos.SnapshotDescription;
43  import org.apache.hadoop.hbase.protobuf.generated.SnapshotProtos.SnapshotRegionManifest;
44  import org.apache.hadoop.hbase.wal.DefaultWALProvider;
45  import org.apache.hadoop.hbase.regionserver.StoreFileInfo;
46  import org.apache.hadoop.hbase.util.FSVisitor;
47  
48  /**
49   * Utility methods for interacting with the snapshot referenced files.
50   */
51  @InterfaceAudience.Private
52  public final class SnapshotReferenceUtil {
53    private static final Log LOG = LogFactory.getLog(SnapshotReferenceUtil.class);
54  
55    public interface StoreFileVisitor {
56      void storeFile(final HRegionInfo regionInfo, final String familyName,
57         final SnapshotRegionManifest.StoreFile storeFile) throws IOException;
58    }
59  
60    public interface SnapshotVisitor extends StoreFileVisitor,
61      FSVisitor.LogFileVisitor {
62    }
63  
64    private SnapshotReferenceUtil() {
65      // private constructor for utility class
66    }
67  
68    /**
69     * Get log directory for a server in a snapshot.
70     *
71     * @param snapshotDir directory where the specific snapshot is stored
72     * @param serverName name of the parent regionserver for the log files
73     * @return path to the log home directory for the archive files.
74     */
75    public static Path getLogsDir(Path snapshotDir, String serverName) {
76      return new Path(snapshotDir, DefaultWALProvider.getWALDirectoryName(serverName));
77    }
78  
79    /**
80     * Iterate over the snapshot store files, restored.edits and logs
81     *
82     * @param conf The current {@link Configuration} instance.
83     * @param fs {@link FileSystem}
84     * @param snapshotDir {@link Path} to the Snapshot directory
85     * @param visitor callback object to get the referenced files
86     * @throws IOException if an error occurred while scanning the directory
87     */
88    public static void visitReferencedFiles(final Configuration conf, final FileSystem fs,
89        final Path snapshotDir, final SnapshotVisitor visitor)
90        throws IOException {
91      SnapshotDescription desc = SnapshotDescriptionUtils.readSnapshotInfo(fs, snapshotDir);
92      visitReferencedFiles(conf, fs, snapshotDir, desc, visitor);
93    }
94  
95    /**
96     * Iterate over the snapshot store files, restored.edits and logs
97     *
98     * @param conf The current {@link Configuration} instance.
99     * @param fs {@link FileSystem}
100    * @param snapshotDir {@link Path} to the Snapshot directory
101    * @param desc the {@link SnapshotDescription} of the snapshot to verify
102    * @param visitor callback object to get the referenced files
103    * @throws IOException if an error occurred while scanning the directory
104    */
105   public static void visitReferencedFiles(final Configuration conf, final FileSystem fs,
106       final Path snapshotDir, final SnapshotDescription desc, final SnapshotVisitor visitor)
107       throws IOException {
108     visitTableStoreFiles(conf, fs, snapshotDir, desc, visitor);
109     visitLogFiles(fs, snapshotDir, visitor);
110   }
111 
112   /**©
113    * Iterate over the snapshot store files
114    *
115    * @param conf The current {@link Configuration} instance.
116    * @param fs {@link FileSystem}
117    * @param snapshotDir {@link Path} to the Snapshot directory
118    * @param desc the {@link SnapshotDescription} of the snapshot to verify
119    * @param visitor callback object to get the store files
120    * @throws IOException if an error occurred while scanning the directory
121    */
122   static void visitTableStoreFiles(final Configuration conf, final FileSystem fs,
123       final Path snapshotDir, final SnapshotDescription desc, final StoreFileVisitor visitor)
124       throws IOException {
125     SnapshotManifest manifest = SnapshotManifest.open(conf, fs, snapshotDir, desc);
126     List<SnapshotRegionManifest> regionManifests = manifest.getRegionManifests();
127     if (regionManifests == null || regionManifests.size() == 0) {
128       LOG.debug("No manifest files present: " + snapshotDir);
129       return;
130     }
131 
132     for (SnapshotRegionManifest regionManifest: regionManifests) {
133       visitRegionStoreFiles(regionManifest, visitor);
134     }
135   }
136 
137   /**
138    * Iterate over the snapshot store files in the specified region
139    *
140    * @param manifest snapshot manifest to inspect
141    * @param visitor callback object to get the store files
142    * @throws IOException if an error occurred while scanning the directory
143    */
144   static void visitRegionStoreFiles(final SnapshotRegionManifest manifest,
145       final StoreFileVisitor visitor) throws IOException {
146     HRegionInfo regionInfo = HRegionInfo.convert(manifest.getRegionInfo());
147     for (SnapshotRegionManifest.FamilyFiles familyFiles: manifest.getFamilyFilesList()) {
148       String familyName = familyFiles.getFamilyName().toStringUtf8();
149       for (SnapshotRegionManifest.StoreFile storeFile: familyFiles.getStoreFilesList()) {
150         visitor.storeFile(regionInfo, familyName, storeFile);
151       }
152     }
153   }
154 
155   /**
156    * Iterate over the snapshot log files
157    *
158    * @param fs {@link FileSystem}
159    * @param snapshotDir {@link Path} to the Snapshot directory
160    * @param visitor callback object to get the log files
161    * @throws IOException if an error occurred while scanning the directory
162    */
163   public static void visitLogFiles(final FileSystem fs, final Path snapshotDir,
164       final FSVisitor.LogFileVisitor visitor) throws IOException {
165     FSVisitor.visitLogFiles(fs, snapshotDir, visitor);
166   }
167 
168   /**
169    * Verify the validity of the snapshot
170    *
171    * @param conf The current {@link Configuration} instance.
172    * @param fs {@link FileSystem}
173    * @param snapshotDir {@link Path} to the Snapshot directory of the snapshot to verify
174    * @param snapshotDesc the {@link SnapshotDescription} of the snapshot to verify
175    * @throws CorruptedSnapshotException if the snapshot is corrupted
176    * @throws IOException if an error occurred while scanning the directory
177    */
178   public static void verifySnapshot(final Configuration conf, final FileSystem fs,
179       final Path snapshotDir, final SnapshotDescription snapshotDesc) throws IOException {
180     SnapshotManifest manifest = SnapshotManifest.open(conf, fs, snapshotDir, snapshotDesc);
181     verifySnapshot(conf, fs, manifest);
182   }
183 
184   /**
185    * Verify the validity of the snapshot
186    *
187    * @param conf The current {@link Configuration} instance.
188    * @param fs {@link FileSystem}
189    * @param manifest snapshot manifest to inspect
190    * @throws CorruptedSnapshotException if the snapshot is corrupted
191    * @throws IOException if an error occurred while scanning the directory
192    */
193   public static void verifySnapshot(final Configuration conf, final FileSystem fs,
194       final SnapshotManifest manifest) throws IOException {
195     final SnapshotDescription snapshotDesc = manifest.getSnapshotDescription();
196     final Path snapshotDir = manifest.getSnapshotDir();
197     concurrentVisitReferencedFiles(conf, fs, manifest, new StoreFileVisitor() {
198       @Override
199       public void storeFile(final HRegionInfo regionInfo, final String family,
200           final SnapshotRegionManifest.StoreFile storeFile) throws IOException {
201         verifyStoreFile(conf, fs, snapshotDir, snapshotDesc, regionInfo, family, storeFile);
202       }
203     });
204   }
205 
206   public static void concurrentVisitReferencedFiles(final Configuration conf, final FileSystem fs,
207       final SnapshotManifest manifest, final StoreFileVisitor visitor) throws IOException {
208     final SnapshotDescription snapshotDesc = manifest.getSnapshotDescription();
209     final Path snapshotDir = manifest.getSnapshotDir();
210 
211     List<SnapshotRegionManifest> regionManifests = manifest.getRegionManifests();
212     if (regionManifests == null || regionManifests.size() == 0) {
213       LOG.debug("No manifest files present: " + snapshotDir);
214       return;
215     }
216 
217     ExecutorService exec = SnapshotManifest.createExecutor(conf, "VerifySnapshot");
218     final ExecutorCompletionService<Void> completionService =
219       new ExecutorCompletionService<Void>(exec);
220     try {
221       for (final SnapshotRegionManifest regionManifest: regionManifests) {
222         completionService.submit(new Callable<Void>() {
223           @Override
224           public Void call() throws IOException {
225             visitRegionStoreFiles(regionManifest, visitor);
226             return null;
227           }
228         });
229       }
230       try {
231         for (int i = 0; i < regionManifests.size(); ++i) {
232           completionService.take().get();
233         }
234       } catch (InterruptedException e) {
235         throw new InterruptedIOException(e.getMessage());
236       } catch (ExecutionException e) {
237         if (e.getCause() instanceof CorruptedSnapshotException) {
238           throw new CorruptedSnapshotException(e.getCause().getMessage(), snapshotDesc);
239         } else {
240           IOException ex = new IOException();
241           ex.initCause(e.getCause());
242           throw ex;
243         }
244       }
245     } finally {
246       exec.shutdown();
247     }
248   }
249 
250   /**
251    * Verify the validity of the snapshot store file
252    *
253    * @param conf The current {@link Configuration} instance.
254    * @param fs {@link FileSystem}
255    * @param snapshotDir {@link Path} to the Snapshot directory of the snapshot to verify
256    * @param snapshot the {@link SnapshotDescription} of the snapshot to verify
257    * @param regionInfo {@link HRegionInfo} of the region that contains the store file
258    * @param family family that contains the store file
259    * @param storeFile the store file to verify
260    * @throws CorruptedSnapshotException if the snapshot is corrupted
261    * @throws IOException if an error occurred while scanning the directory
262    */
263   private static void verifyStoreFile(final Configuration conf, final FileSystem fs,
264       final Path snapshotDir, final SnapshotDescription snapshot, final HRegionInfo regionInfo,
265       final String family, final SnapshotRegionManifest.StoreFile storeFile) throws IOException {
266     TableName table = TableName.valueOf(snapshot.getTable());
267     String fileName = storeFile.getName();
268 
269     Path refPath = null;
270     if (StoreFileInfo.isReference(fileName)) {
271       // If is a reference file check if the parent file is present in the snapshot
272       refPath = new Path(new Path(regionInfo.getEncodedName(), family), fileName);
273       refPath = StoreFileInfo.getReferredToFile(refPath);
274       String refRegion = refPath.getParent().getParent().getName();
275       refPath = HFileLink.createPath(table, refRegion, family, refPath.getName());
276       if (!HFileLink.buildFromHFileLinkPattern(conf, refPath).exists(fs)) {
277         throw new CorruptedSnapshotException("Missing parent hfile for: " + fileName +
278           " path=" + refPath, snapshot);
279       }
280 
281       if (storeFile.hasReference()) {
282         // We don't really need to look for the file on-disk
283         // we already have the Reference information embedded here.
284         return;
285       }
286     }
287 
288     Path linkPath;
289     if (refPath != null && HFileLink.isHFileLink(refPath)) {
290       linkPath = new Path(family, refPath.getName());
291     } else if (HFileLink.isHFileLink(fileName)) {
292       linkPath = new Path(family, fileName);
293     } else {
294       linkPath = new Path(family, HFileLink.createHFileLinkName(
295               table, regionInfo.getEncodedName(), fileName));
296     }
297 
298     // check if the linked file exists (in the archive, or in the table dir)
299     HFileLink link = HFileLink.buildFromHFileLinkPattern(conf, linkPath);
300     try {
301       FileStatus fstat = link.getFileStatus(fs);
302       if (storeFile.hasFileSize() && storeFile.getFileSize() != fstat.getLen()) {
303         String msg = "hfile: " + fileName + " size does not match with the expected one. " +
304           " found=" + fstat.getLen() + " expected=" + storeFile.getFileSize();
305         LOG.error(msg);
306         throw new CorruptedSnapshotException(msg, snapshot);
307       }
308     } catch (FileNotFoundException e) {
309       String msg = "Can't find hfile: " + fileName + " in the real (" +
310           link.getOriginPath() + ") or archive (" + link.getArchivePath()
311           + ") directory for the primary table.";
312       LOG.error(msg);
313       throw new CorruptedSnapshotException(msg, snapshot);
314     }
315   }
316 
317   /**
318    * Returns the store file names in the snapshot.
319    *
320    * @param conf The current {@link Configuration} instance.
321    * @param fs {@link FileSystem}
322    * @param snapshotDir {@link Path} to the Snapshot directory
323    * @throws IOException if an error occurred while scanning the directory
324    * @return the names of hfiles in the specified snaphot
325    */
326   public static Set<String> getHFileNames(final Configuration conf, final FileSystem fs,
327       final Path snapshotDir) throws IOException {
328     SnapshotDescription desc = SnapshotDescriptionUtils.readSnapshotInfo(fs, snapshotDir);
329     return getHFileNames(conf, fs, snapshotDir, desc);
330   }
331 
332   /**
333    * Returns the store file names in the snapshot.
334    *
335    * @param conf The current {@link Configuration} instance.
336    * @param fs {@link FileSystem}
337    * @param snapshotDir {@link Path} to the Snapshot directory
338    * @param snapshotDesc the {@link SnapshotDescription} of the snapshot to inspect
339    * @throws IOException if an error occurred while scanning the directory
340    * @return the names of hfiles in the specified snaphot
341    */
342   private static Set<String> getHFileNames(final Configuration conf, final FileSystem fs,
343       final Path snapshotDir, final SnapshotDescription snapshotDesc)
344       throws IOException {
345     final Set<String> names = new HashSet<String>();
346     visitTableStoreFiles(conf, fs, snapshotDir, snapshotDesc, new StoreFileVisitor() {
347       @Override
348       public void storeFile(final HRegionInfo regionInfo, final String family,
349             final SnapshotRegionManifest.StoreFile storeFile) throws IOException {
350         String hfile = storeFile.getName();
351         if (HFileLink.isHFileLink(hfile)) {
352           names.add(HFileLink.getReferencedHFileName(hfile));
353         } else {
354           names.add(hfile);
355         }
356       }
357     });
358     return names;
359   }
360 
361   /**
362    * Returns the log file names available in the snapshot.
363    *
364    * @param fs {@link FileSystem}
365    * @param snapshotDir {@link Path} to the Snapshot directory
366    * @throws IOException if an error occurred while scanning the directory
367    * @return the names of wals in the specified snaphot
368    */
369   public static Set<String> getWALNames(final FileSystem fs, final Path snapshotDir)
370       throws IOException {
371     final Set<String> names = new HashSet<String>();
372     visitLogFiles(fs, snapshotDir, new FSVisitor.LogFileVisitor() {
373       @Override
374       public void logFile (final String server, final String logfile) throws IOException {
375         names.add(logfile);
376       }
377     });
378     return names;
379   }
380 }