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.FileNotFoundException;
021import java.io.IOException;
022import java.io.InterruptedIOException;
023import java.util.HashSet;
024import java.util.List;
025import java.util.Set;
026import java.util.concurrent.Callable;
027import java.util.concurrent.ExecutionException;
028import java.util.concurrent.ExecutorCompletionService;
029import java.util.concurrent.ExecutorService;
030import org.apache.hadoop.conf.Configuration;
031import org.apache.hadoop.fs.FileStatus;
032import org.apache.hadoop.fs.FileSystem;
033import org.apache.hadoop.fs.Path;
034import org.apache.hadoop.hbase.TableName;
035import org.apache.hadoop.hbase.client.RegionInfo;
036import org.apache.hadoop.hbase.io.HFileLink;
037import org.apache.hadoop.hbase.mob.MobUtils;
038import org.apache.hadoop.hbase.regionserver.StoreFileInfo;
039import org.apache.hadoop.hbase.util.HFileArchiveUtil;
040import org.apache.yetus.audience.InterfaceAudience;
041import org.slf4j.Logger;
042import org.slf4j.LoggerFactory;
043
044import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
045import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription;
046import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotRegionManifest;
047
048/**
049 * Utility methods for interacting with the snapshot referenced files.
050 */
051@InterfaceAudience.Private
052public final class SnapshotReferenceUtil {
053  private static final Logger LOG = LoggerFactory.getLogger(SnapshotReferenceUtil.class);
054
055  public interface StoreFileVisitor {
056    void storeFile(final RegionInfo regionInfo, final String familyName,
057      final SnapshotRegionManifest.StoreFile storeFile) throws IOException;
058  }
059
060  public interface SnapshotVisitor extends StoreFileVisitor {
061  }
062
063  private SnapshotReferenceUtil() {
064    // private constructor for utility class
065  }
066
067  /**
068   * Iterate over the snapshot store files
069   * @param conf        The current {@link Configuration} instance.
070   * @param fs          {@link FileSystem}
071   * @param snapshotDir {@link Path} to the Snapshot directory
072   * @param visitor     callback object to get the referenced files
073   * @throws IOException if an error occurred while scanning the directory
074   */
075  public static void visitReferencedFiles(final Configuration conf, final FileSystem fs,
076    final Path snapshotDir, final SnapshotVisitor visitor) throws IOException {
077    SnapshotDescription desc = SnapshotDescriptionUtils.readSnapshotInfo(fs, snapshotDir);
078    visitReferencedFiles(conf, fs, snapshotDir, desc, visitor);
079  }
080
081  /**
082   * Iterate over the snapshot store files, restored.edits and logs
083   * @param conf        The current {@link Configuration} instance.
084   * @param fs          {@link FileSystem}
085   * @param snapshotDir {@link Path} to the Snapshot directory
086   * @param desc        the {@link SnapshotDescription} of the snapshot to verify
087   * @param visitor     callback object to get the referenced files
088   * @throws IOException if an error occurred while scanning the directory
089   */
090  public static void visitReferencedFiles(final Configuration conf, final FileSystem fs,
091    final Path snapshotDir, final SnapshotDescription desc, final SnapshotVisitor visitor)
092    throws IOException {
093    visitTableStoreFiles(conf, fs, snapshotDir, desc, visitor);
094  }
095
096  /**
097   * © Iterate over the snapshot store files
098   * @param conf        The current {@link Configuration} instance.
099   * @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 store files
103   * @throws IOException if an error occurred while scanning the directory
104   */
105  static void visitTableStoreFiles(final Configuration conf, final FileSystem fs,
106    final Path snapshotDir, final SnapshotDescription desc, final StoreFileVisitor visitor)
107    throws IOException {
108    SnapshotManifest manifest = SnapshotManifest.open(conf, fs, snapshotDir, desc);
109    List<SnapshotRegionManifest> regionManifests = manifest.getRegionManifests();
110    if (regionManifests == null || regionManifests.isEmpty()) {
111      LOG.debug("No manifest files present: " + snapshotDir);
112      return;
113    }
114
115    for (SnapshotRegionManifest regionManifest : regionManifests) {
116      visitRegionStoreFiles(regionManifest, visitor);
117    }
118  }
119
120  /**
121   * Iterate over the snapshot store files in the specified region
122   * @param manifest snapshot manifest to inspect
123   * @param visitor  callback object to get the store files
124   * @throws IOException if an error occurred while scanning the directory
125   */
126  static void visitRegionStoreFiles(final SnapshotRegionManifest manifest,
127    final StoreFileVisitor visitor) throws IOException {
128    RegionInfo regionInfo = ProtobufUtil.toRegionInfo(manifest.getRegionInfo());
129    for (SnapshotRegionManifest.FamilyFiles familyFiles : manifest.getFamilyFilesList()) {
130      String familyName = familyFiles.getFamilyName().toStringUtf8();
131      for (SnapshotRegionManifest.StoreFile storeFile : familyFiles.getStoreFilesList()) {
132        visitor.storeFile(regionInfo, familyName, storeFile);
133      }
134    }
135  }
136
137  /**
138   * Verify the validity of the snapshot
139   * @param conf         The current {@link Configuration} instance.
140   * @param fs           {@link FileSystem}
141   * @param snapshotDir  {@link Path} to the Snapshot directory of the snapshot to verify
142   * @param snapshotDesc the {@link SnapshotDescription} of the snapshot to verify
143   * @throws CorruptedSnapshotException if the snapshot is corrupted
144   * @throws IOException                if an error occurred while scanning the directory
145   */
146  public static void verifySnapshot(final Configuration conf, final FileSystem fs,
147    final Path snapshotDir, final SnapshotDescription snapshotDesc) throws IOException {
148    SnapshotManifest manifest = SnapshotManifest.open(conf, fs, snapshotDir, snapshotDesc);
149    verifySnapshot(conf, fs, manifest);
150  }
151
152  /**
153   * Verify the validity of the snapshot
154   * @param conf     The current {@link Configuration} instance.
155   * @param fs       {@link FileSystem}
156   * @param manifest snapshot manifest to inspect
157   * @throws CorruptedSnapshotException if the snapshot is corrupted
158   * @throws IOException                if an error occurred while scanning the directory
159   */
160  public static void verifySnapshot(final Configuration conf, final FileSystem fs,
161    final SnapshotManifest manifest) throws IOException {
162    final SnapshotDescription snapshotDesc = manifest.getSnapshotDescription();
163    final Path snapshotDir = manifest.getSnapshotDir();
164    concurrentVisitReferencedFiles(conf, fs, manifest, "VerifySnapshot", new StoreFileVisitor() {
165      @Override
166      public void storeFile(final RegionInfo regionInfo, final String family,
167        final SnapshotRegionManifest.StoreFile storeFile) throws IOException {
168        verifyStoreFile(conf, fs, snapshotDir, snapshotDesc, regionInfo, family, storeFile);
169      }
170    });
171  }
172
173  public static void concurrentVisitReferencedFiles(final Configuration conf, final FileSystem fs,
174    final SnapshotManifest manifest, final String desc, final StoreFileVisitor visitor)
175    throws IOException {
176
177    final Path snapshotDir = manifest.getSnapshotDir();
178    List<SnapshotRegionManifest> regionManifests = manifest.getRegionManifests();
179    if (regionManifests == null || regionManifests.isEmpty()) {
180      LOG.debug("No manifest files present: " + snapshotDir);
181      return;
182    }
183
184    ExecutorService exec = SnapshotManifest.createExecutor(conf, desc);
185
186    try {
187      concurrentVisitReferencedFiles(conf, fs, manifest, exec, visitor);
188    } finally {
189      exec.shutdown();
190    }
191  }
192
193  public static void concurrentVisitReferencedFiles(final Configuration conf, final FileSystem fs,
194    final SnapshotManifest manifest, final ExecutorService exec, final StoreFileVisitor visitor)
195    throws IOException {
196    final SnapshotDescription snapshotDesc = manifest.getSnapshotDescription();
197    final Path snapshotDir = manifest.getSnapshotDir();
198
199    List<SnapshotRegionManifest> regionManifests = manifest.getRegionManifests();
200    if (regionManifests == null || regionManifests.isEmpty()) {
201      LOG.debug("No manifest files present: " + snapshotDir);
202      return;
203    }
204
205    final ExecutorCompletionService<Void> completionService = new ExecutorCompletionService<>(exec);
206
207    for (final SnapshotRegionManifest regionManifest : regionManifests) {
208      completionService.submit(new Callable<Void>() {
209        @Override
210        public Void call() throws IOException {
211          visitRegionStoreFiles(regionManifest, visitor);
212          return null;
213        }
214      });
215    }
216    try {
217      for (int i = 0; i < regionManifests.size(); ++i) {
218        completionService.take().get();
219      }
220    } catch (InterruptedException e) {
221      throw new InterruptedIOException(e.getMessage());
222    } catch (ExecutionException e) {
223      if (e.getCause() instanceof CorruptedSnapshotException) {
224        throw new CorruptedSnapshotException(e.getCause().getMessage(),
225          ProtobufUtil.createSnapshotDesc(snapshotDesc));
226      } else {
227        throw new IOException(e.getCause());
228      }
229    }
230  }
231
232  /**
233   * Verify the validity of the snapshot store file
234   * @param conf        The current {@link Configuration} instance.
235   * @param fs          {@link FileSystem}
236   * @param snapshotDir {@link Path} to the Snapshot directory of the snapshot to verify
237   * @param snapshot    the {@link SnapshotDescription} of the snapshot to verify
238   * @param regionInfo  {@link RegionInfo} of the region that contains the store file
239   * @param family      family that contains the store file
240   * @param storeFile   the store file to verify
241   * @throws CorruptedSnapshotException if the snapshot is corrupted
242   * @throws IOException                if an error occurred while scanning the directory
243   */
244  private static void verifyStoreFile(final Configuration conf, final FileSystem fs,
245    final Path snapshotDir, final SnapshotDescription snapshot, final RegionInfo regionInfo,
246    final String family, final SnapshotRegionManifest.StoreFile storeFile) throws IOException {
247    TableName table = TableName.valueOf(snapshot.getTable());
248    String fileName = storeFile.getName();
249
250    Path refPath = null;
251    if (StoreFileInfo.isReference(fileName)) {
252      // If is a reference file check if the parent file is present in the snapshot
253      refPath = new Path(new Path(regionInfo.getEncodedName(), family), fileName);
254      refPath = StoreFileInfo.getReferredToFile(refPath);
255      String refRegion = refPath.getParent().getParent().getName();
256      refPath = HFileLink.createPath(table, refRegion, family, refPath.getName());
257      if (!HFileLink.buildFromHFileLinkPattern(conf, refPath).exists(fs)) {
258        throw new CorruptedSnapshotException(
259          "Missing parent hfile for: " + fileName + " path=" + refPath,
260          ProtobufUtil.createSnapshotDesc(snapshot));
261      }
262
263      if (storeFile.hasReference()) {
264        // We don't really need to look for the file on-disk
265        // we already have the Reference information embedded here.
266        return;
267      }
268    }
269
270    Path linkPath;
271    if (refPath != null && HFileLink.isHFileLink(refPath)) {
272      linkPath = new Path(family, refPath.getName());
273    } else if (HFileLink.isHFileLink(fileName)) {
274      linkPath = new Path(family, fileName);
275    } else {
276      linkPath = new Path(family,
277        HFileLink.createHFileLinkName(table, regionInfo.getEncodedName(), fileName));
278    }
279
280    // check if the linked file exists (in the archive, or in the table dir)
281    HFileLink link = null;
282    if (MobUtils.isMobRegionInfo(regionInfo)) {
283      // for mob region
284      link = HFileLink.buildFromHFileLinkPattern(MobUtils.getQualifiedMobRootDir(conf),
285        HFileArchiveUtil.getArchivePath(conf), linkPath);
286    } else {
287      // not mob region
288      link = HFileLink.buildFromHFileLinkPattern(conf, linkPath);
289    }
290    try {
291      FileStatus fstat = link.getFileStatus(fs);
292      if (storeFile.hasFileSize() && storeFile.getFileSize() != fstat.getLen()) {
293        String msg = "hfile: " + fileName + " size does not match with the expected one. "
294          + " found=" + fstat.getLen() + " expected=" + storeFile.getFileSize();
295        LOG.error(msg);
296        throw new CorruptedSnapshotException(msg, ProtobufUtil.createSnapshotDesc(snapshot));
297      }
298    } catch (FileNotFoundException e) {
299      String msg = "Can't find hfile: " + fileName + " in the real (" + link.getOriginPath()
300        + ") or archive (" + link.getArchivePath() + ") directory for the primary table.";
301      LOG.error(msg);
302      throw new CorruptedSnapshotException(msg, ProtobufUtil.createSnapshotDesc(snapshot));
303    }
304  }
305
306  /**
307   * Returns the store file names in the snapshot.
308   * @param conf        The current {@link Configuration} instance.
309   * @param fs          {@link FileSystem}
310   * @param snapshotDir {@link Path} to the Snapshot directory
311   * @throws IOException if an error occurred while scanning the directory
312   * @return the names of hfiles in the specified snaphot
313   */
314  public static Set<String> getHFileNames(final Configuration conf, final FileSystem fs,
315    final Path snapshotDir) throws IOException {
316    SnapshotDescription desc = SnapshotDescriptionUtils.readSnapshotInfo(fs, snapshotDir);
317    return getHFileNames(conf, fs, snapshotDir, desc);
318  }
319
320  /**
321   * Returns the store file names in the snapshot.
322   * @param conf         The current {@link Configuration} instance.
323   * @param fs           {@link FileSystem}
324   * @param snapshotDir  {@link Path} to the Snapshot directory
325   * @param snapshotDesc the {@link SnapshotDescription} of the snapshot to inspect
326   * @throws IOException if an error occurred while scanning the directory
327   * @return the names of hfiles in the specified snaphot
328   */
329  private static Set<String> getHFileNames(final Configuration conf, final FileSystem fs,
330    final Path snapshotDir, final SnapshotDescription snapshotDesc) throws IOException {
331    final Set<String> names = new HashSet<>();
332    visitTableStoreFiles(conf, fs, snapshotDir, snapshotDesc, new StoreFileVisitor() {
333      @Override
334      public void storeFile(final RegionInfo regionInfo, final String family,
335        final SnapshotRegionManifest.StoreFile storeFile) throws IOException {
336        String hfile = storeFile.getName();
337        if (HFileLink.isHFileLink(hfile)) {
338          names.add(HFileLink.getReferencedHFileName(hfile));
339        } else if (StoreFileInfo.isReference(hfile)) {
340          Path refPath =
341            StoreFileInfo.getReferredToFile(new Path(new Path(
342              new Path(new Path(regionInfo.getTable().getNamespaceAsString(),
343                regionInfo.getTable().getQualifierAsString()), regionInfo.getEncodedName()),
344              family), hfile));
345          names.add(hfile);
346          names.add(refPath.getName());
347          if (HFileLink.isHFileLink(refPath.getName())) {
348            names.add(HFileLink.getReferencedHFileName(refPath.getName()));
349          }
350        } else {
351          names.add(hfile);
352        }
353      }
354    });
355    return names;
356  }
357}