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  package org.apache.hadoop.hbase.master.snapshot;
19  
20  import java.io.IOException;
21  import java.util.List;
22  import java.util.Map;
23  import java.util.Set;
24  
25  import org.apache.commons.logging.Log;
26  import org.apache.commons.logging.LogFactory;
27  import org.apache.hadoop.hbase.classification.InterfaceAudience;
28  import org.apache.hadoop.hbase.classification.InterfaceStability;
29  import org.apache.hadoop.fs.FileSystem;
30  import org.apache.hadoop.fs.Path;
31  import org.apache.hadoop.hbase.TableName;
32  import org.apache.hadoop.hbase.HRegionInfo;
33  import org.apache.hadoop.hbase.HTableDescriptor;
34  import org.apache.hadoop.hbase.client.RegionReplicaUtil;
35  import org.apache.hadoop.hbase.MetaTableAccessor;
36  import org.apache.hadoop.hbase.master.MasterServices;
37  import org.apache.hadoop.hbase.protobuf.generated.HBaseProtos.SnapshotDescription;
38  import org.apache.hadoop.hbase.protobuf.generated.SnapshotProtos.SnapshotRegionManifest;
39  import org.apache.hadoop.hbase.snapshot.ClientSnapshotDescriptionUtils;
40  import org.apache.hadoop.hbase.snapshot.CorruptedSnapshotException;
41  import org.apache.hadoop.hbase.snapshot.SnapshotDescriptionUtils;
42  import org.apache.hadoop.hbase.snapshot.SnapshotManifest;
43  import org.apache.hadoop.hbase.snapshot.SnapshotReferenceUtil;
44  import org.apache.hadoop.hbase.zookeeper.MetaTableLocator;
45  
46  /**
47   * General snapshot verification on the master.
48   * <p>
49   * This is a light-weight verification mechanism for all the files in a snapshot. It doesn't
50   * attempt to verify that the files are exact copies (that would be paramount to taking the
51   * snapshot again!), but instead just attempts to ensure that the files match the expected
52   * files and are the same length.
53   * <p>
54   * Taking an online snapshots can race against other operations and this is an last line of
55   * defense.  For example, if meta changes between when snapshots are taken not all regions of a
56   * table may be present.  This can be caused by a region split (daughters present on this scan,
57   * but snapshot took parent), or move (snapshots only checks lists of region servers, a move could
58   * have caused a region to be skipped or done twice).
59   * <p>
60   * Current snapshot files checked:
61   * <ol>
62   * <li>SnapshotDescription is readable</li>
63   * <li>Table info is readable</li>
64   * <li>Regions</li>
65   * </ol>
66   * <ul>
67   * <li>Matching regions in the snapshot as currently in the table</li>
68   * <li>{@link HRegionInfo} matches the current and stored regions</li>
69   * <li>All referenced hfiles have valid names</li>
70   * <li>All the hfiles are present (either in .archive directory in the region)</li>
71   * <li>All recovered.edits files are present (by name) and have the correct file size</li>
72   * </ul>
73   */
74  @InterfaceAudience.Private
75  @InterfaceStability.Unstable
76  public final class MasterSnapshotVerifier {
77    private static final Log LOG = LogFactory.getLog(MasterSnapshotVerifier.class);
78  
79    private SnapshotDescription snapshot;
80    private FileSystem fs;
81    private Path rootDir;
82    private TableName tableName;
83    private MasterServices services;
84  
85    /**
86     * @param services services for the master
87     * @param snapshot snapshot to check
88     * @param rootDir root directory of the hbase installation.
89     */
90    public MasterSnapshotVerifier(MasterServices services, SnapshotDescription snapshot, Path rootDir) {
91      this.fs = services.getMasterFileSystem().getFileSystem();
92      this.services = services;
93      this.snapshot = snapshot;
94      this.rootDir = rootDir;
95      this.tableName = TableName.valueOf(snapshot.getTable());
96    }
97  
98    /**
99     * Verify that the snapshot in the directory is a valid snapshot
100    * @param snapshotDir snapshot directory to check
101    * @param snapshotServers {@link org.apache.hadoop.hbase.ServerName} 
102    * of the servers that are involved in the snapshot
103    * @throws CorruptedSnapshotException if the snapshot is invalid
104    * @throws IOException if there is an unexpected connection issue to the filesystem
105    */
106   public void verifySnapshot(Path snapshotDir, Set<String> snapshotServers)
107       throws CorruptedSnapshotException, IOException {
108     SnapshotManifest manifest = SnapshotManifest.open(services.getConfiguration(), fs,
109                                                       snapshotDir, snapshot);
110     // verify snapshot info matches
111     verifySnapshotDescription(snapshotDir);
112 
113     // check that tableinfo is a valid table description
114     verifyTableInfo(manifest);
115 
116     // check that each region is valid
117     verifyRegions(manifest);
118   }
119 
120   /**
121    * Check that the snapshot description written in the filesystem matches the current snapshot
122    * @param snapshotDir snapshot directory to check
123    */
124   private void verifySnapshotDescription(Path snapshotDir) throws CorruptedSnapshotException {
125     SnapshotDescription found = SnapshotDescriptionUtils.readSnapshotInfo(fs, snapshotDir);
126     if (!this.snapshot.equals(found)) {
127       throw new CorruptedSnapshotException("Snapshot read (" + found
128           + ") doesn't equal snapshot we ran (" + snapshot + ").", snapshot);
129     }
130   }
131 
132   /**
133    * Check that the table descriptor for the snapshot is a valid table descriptor
134    * @param manifest snapshot manifest to inspect
135    */
136   private void verifyTableInfo(final SnapshotManifest manifest) throws IOException {
137     HTableDescriptor htd = manifest.getTableDescriptor();
138     if (htd == null) {
139       throw new CorruptedSnapshotException("Missing Table Descriptor", snapshot);
140     }
141 
142     if (!htd.getNameAsString().equals(snapshot.getTable())) {
143       throw new CorruptedSnapshotException("Invalid Table Descriptor. Expected "
144         + snapshot.getTable() + " name, got " + htd.getNameAsString(), snapshot);
145     }
146   }
147 
148   /**
149    * Check that all the regions in the snapshot are valid, and accounted for.
150    * @param manifest snapshot manifest to inspect
151    * @throws IOException if we can't reach hbase:meta or read the files from the FS
152    */
153   private void verifyRegions(final SnapshotManifest manifest) throws IOException {
154     List<HRegionInfo> regions;
155     if (TableName.META_TABLE_NAME.equals(tableName)) {
156       regions = new MetaTableLocator().getMetaRegions(services.getZooKeeper());
157     } else {
158       regions = MetaTableAccessor.getTableRegions(services.getZooKeeper(),
159         services.getConnection(), tableName);
160     }
161     // Remove the non-default regions
162     RegionReplicaUtil.removeNonDefaultRegions(regions);
163 
164     Map<String, SnapshotRegionManifest> regionManifests = manifest.getRegionManifestsMap();
165     if (regionManifests == null) {
166       String msg = "Snapshot " + ClientSnapshotDescriptionUtils.toString(snapshot) + " looks empty";
167       LOG.error(msg);
168       throw new CorruptedSnapshotException(msg);
169     }
170 
171     String errorMsg = "";
172     if (regionManifests.size() != regions.size()) {
173       errorMsg = "Regions moved during the snapshot '" +
174                    ClientSnapshotDescriptionUtils.toString(snapshot) + "'. expected=" +
175                    regions.size() + " snapshotted=" + regionManifests.size() + ".";
176       LOG.error(errorMsg);
177     }
178 
179     // Verify HRegionInfo
180     for (HRegionInfo region : regions) {
181       SnapshotRegionManifest regionManifest = regionManifests.get(region.getEncodedName());
182       if (regionManifest == null) {
183         // could happen due to a move or split race.
184         String mesg = " No snapshot region directory found for region:" + region;
185         if (errorMsg.isEmpty()) errorMsg = mesg;
186         LOG.error(mesg);
187         continue;
188       }
189 
190       verifyRegionInfo(region, regionManifest);
191     }
192 
193     if (!errorMsg.isEmpty()) {
194       throw new CorruptedSnapshotException(errorMsg);
195     }
196 
197     // Verify Snapshot HFiles
198     SnapshotReferenceUtil.verifySnapshot(services.getConfiguration(), fs, manifest);
199   }
200 
201   /**
202    * Verify that the regionInfo is valid
203    * @param region the region to check
204    * @param manifest snapshot manifest to inspect
205    */
206   private void verifyRegionInfo(final HRegionInfo region,
207       final SnapshotRegionManifest manifest) throws IOException {
208     HRegionInfo manifestRegionInfo = HRegionInfo.convert(manifest.getRegionInfo());
209     if (!region.equals(manifestRegionInfo)) {
210       String msg = "Manifest region info " + manifestRegionInfo +
211                    "doesn't match expected region:" + region;
212       throw new CorruptedSnapshotException(msg, snapshot);
213     }
214   }
215 }