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.IOException;
021import java.net.URI;
022import java.security.PrivilegedExceptionAction;
023import java.util.Collections;
024import java.util.concurrent.TimeUnit;
025import org.apache.hadoop.conf.Configuration;
026import org.apache.hadoop.fs.FSDataInputStream;
027import org.apache.hadoop.fs.FSDataOutputStream;
028import org.apache.hadoop.fs.FileSystem;
029import org.apache.hadoop.fs.FileUtil;
030import org.apache.hadoop.fs.Path;
031import org.apache.hadoop.fs.permission.FsPermission;
032import org.apache.hadoop.hbase.HConstants;
033import org.apache.hadoop.hbase.TableName;
034import org.apache.hadoop.hbase.client.Admin;
035import org.apache.hadoop.hbase.client.Connection;
036import org.apache.hadoop.hbase.client.ConnectionFactory;
037import org.apache.hadoop.hbase.security.User;
038import org.apache.hadoop.hbase.security.access.PermissionStorage;
039import org.apache.hadoop.hbase.security.access.ShadedAccessControlUtil;
040import org.apache.hadoop.hbase.security.access.UserPermission;
041import org.apache.hadoop.hbase.util.CommonFSUtils;
042import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
043import org.apache.hadoop.hbase.util.FSUtils;
044import org.apache.yetus.audience.InterfaceAudience;
045import org.slf4j.Logger;
046import org.slf4j.LoggerFactory;
047
048import org.apache.hbase.thirdparty.com.google.common.collect.ListMultimap;
049
050import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
051import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription;
052
053/**
054 * Utility class to help manage {@link SnapshotDescription SnapshotDesriptions}.
055 * <p>
056 * Snapshots are laid out on disk like this:
057 *
058 * <pre>
059 * /hbase/.snapshots
060 *          /.tmp                &lt;---- working directory
061 *          /[snapshot name]     &lt;----- completed snapshot
062 * </pre>
063 *
064 * A completed snapshot named 'completed' then looks like (multiple regions, servers, files, etc.
065 * signified by '...' on the same directory depth).
066 *
067 * <pre>
068 * /hbase/.snapshots/completed
069 *                   .snapshotinfo          &lt;--- Description of the snapshot
070 *                   .tableinfo             &lt;--- Copy of the tableinfo
071 *                    /.logs
072 *                        /[server_name]
073 *                            /... [log files]
074 *                         ...
075 *                   /[region name]           &lt;---- All the region's information
076 *                   .regioninfo              &lt;---- Copy of the HRegionInfo
077 *                      /[column family name]
078 *                          /[hfile name]     &lt;--- name of the hfile in the real region
079 *                          ...
080 *                      ...
081 *                    ...
082 * </pre>
083 *
084 * Utility methods in this class are useful for getting the correct locations for different parts of
085 * the snapshot, as well as moving completed snapshots into place (see
086 * {@link #completeSnapshot}, and writing the
087 * {@link SnapshotDescription} to the working snapshot directory.
088 */
089@InterfaceAudience.Private
090public final class SnapshotDescriptionUtils {
091
092  /**
093   * Filter that only accepts completed snapshot directories
094   */
095  public static class CompletedSnaphotDirectoriesFilter extends FSUtils.BlackListDirFilter {
096
097    /**
098     * @param fs
099     */
100    public CompletedSnaphotDirectoriesFilter(FileSystem fs) {
101      super(fs, Collections.singletonList(SNAPSHOT_TMP_DIR_NAME));
102    }
103  }
104
105  private static final Logger LOG = LoggerFactory.getLogger(SnapshotDescriptionUtils.class);
106  /**
107   * Version of the fs layout for a snapshot. Future snapshots may have different file layouts,
108   * which we may need to read in differently.
109   */
110  public static final int SNAPSHOT_LAYOUT_VERSION = SnapshotManifestV2.DESCRIPTOR_VERSION;
111
112  // snapshot directory constants
113  /**
114   * The file contains the snapshot basic information and it is under the directory of a snapshot.
115   */
116  public static final String SNAPSHOTINFO_FILE = ".snapshotinfo";
117
118  /** Temporary directory under the snapshot directory to store in-progress snapshots */
119  public static final String SNAPSHOT_TMP_DIR_NAME = ".tmp";
120
121  /**
122   * The configuration property that determines the filepath of the snapshot
123   * base working directory
124   */
125  public static final String SNAPSHOT_WORKING_DIR = "hbase.snapshot.working.dir";
126
127  // snapshot operation values
128  /** Default value if no start time is specified */
129  public static final long NO_SNAPSHOT_START_TIME_SPECIFIED = 0;
130
131  // Default value if no ttl is specified for Snapshot
132  private static final long NO_SNAPSHOT_TTL_SPECIFIED = 0;
133
134  public static final String MASTER_SNAPSHOT_TIMEOUT_MILLIS = "hbase.snapshot.master.timeout.millis";
135
136  /** By default, wait 300 seconds for a snapshot to complete */
137  public static final long DEFAULT_MAX_WAIT_TIME = 60000 * 5 ;
138
139
140  /**
141   * By default, check to see if the snapshot is complete (ms)
142   * @deprecated Use {@link #DEFAULT_MAX_WAIT_TIME} instead.
143   * */
144  @Deprecated
145  public static final int SNAPSHOT_TIMEOUT_MILLIS_DEFAULT = 60000 * 5;
146
147  /**
148   * Conf key for # of ms elapsed before injecting a snapshot timeout error when waiting for
149   * completion.
150   * @deprecated Use {@link #MASTER_SNAPSHOT_TIMEOUT_MILLIS} instead.
151   */
152  @Deprecated
153  public static final String SNAPSHOT_TIMEOUT_MILLIS_KEY = "hbase.snapshot.master.timeoutMillis";
154
155  private SnapshotDescriptionUtils() {
156    // private constructor for utility class
157  }
158
159  /**
160   * @param conf {@link Configuration} from which to check for the timeout
161   * @param type type of snapshot being taken
162   * @param defaultMaxWaitTime Default amount of time to wait, if none is in the configuration
163   * @return the max amount of time the master should wait for a snapshot to complete
164   */
165  public static long getMaxMasterTimeout(Configuration conf, SnapshotDescription.Type type,
166      long defaultMaxWaitTime) {
167    String confKey;
168    switch (type) {
169    case DISABLED:
170    default:
171      confKey = MASTER_SNAPSHOT_TIMEOUT_MILLIS;
172    }
173    return Math.max(conf.getLong(confKey, defaultMaxWaitTime),
174        conf.getLong(SNAPSHOT_TIMEOUT_MILLIS_KEY, defaultMaxWaitTime));
175  }
176
177  /**
178   * Get the snapshot root directory. All the snapshots are kept under this directory, i.e.
179   * ${hbase.rootdir}/.snapshot
180   * @param rootDir hbase root directory
181   * @return the base directory in which all snapshots are kept
182   */
183  public static Path getSnapshotRootDir(final Path rootDir) {
184    return new Path(rootDir, HConstants.SNAPSHOT_DIR_NAME);
185  }
186
187  /**
188   * Get the directory for a specified snapshot. This directory is a sub-directory of snapshot root
189   * directory and all the data files for a snapshot are kept under this directory.
190   * @param snapshot snapshot being taken
191   * @param rootDir hbase root directory
192   * @return the final directory for the completed snapshot
193   */
194  public static Path getCompletedSnapshotDir(final SnapshotDescription snapshot, final Path rootDir) {
195    return getCompletedSnapshotDir(snapshot.getName(), rootDir);
196  }
197
198  /**
199   * Get the directory for a completed snapshot. This directory is a sub-directory of snapshot root
200   * directory and all the data files for a snapshot are kept under this directory.
201   * @param snapshotName name of the snapshot being taken
202   * @param rootDir hbase root directory
203   * @return the final directory for the completed snapshot
204   */
205  public static Path getCompletedSnapshotDir(final String snapshotName, final Path rootDir) {
206    return getSpecifiedSnapshotDir(getSnapshotsDir(rootDir), snapshotName);
207  }
208
209  /**
210   * Get the general working directory for snapshots - where they are built, where they are
211   * temporarily copied on export, etc.
212   * @param rootDir root directory of the HBase installation
213   * @param conf Configuration of the HBase instance
214   * @return Path to the snapshot tmp directory, relative to the passed root directory
215   */
216  public static Path getWorkingSnapshotDir(final Path rootDir, final Configuration conf) {
217    return new Path(conf.get(SNAPSHOT_WORKING_DIR,
218        getDefaultWorkingSnapshotDir(rootDir).toString()));
219  }
220
221  /**
222   * Get the directory to build a snapshot, before it is finalized
223   * @param snapshot snapshot that will be built
224   * @param rootDir root directory of the hbase installation
225   * @param conf Configuration of the HBase instance
226   * @return {@link Path} where one can build a snapshot
227   */
228  public static Path getWorkingSnapshotDir(SnapshotDescription snapshot, final Path rootDir,
229      Configuration conf) {
230    return getWorkingSnapshotDir(snapshot.getName(), rootDir, conf);
231  }
232
233  /**
234   * Get the directory to build a snapshot, before it is finalized
235   * @param snapshotName name of the snapshot
236   * @param rootDir root directory of the hbase installation
237   * @param conf Configuration of the HBase instance
238   * @return {@link Path} where one can build a snapshot
239   */
240  public static Path getWorkingSnapshotDir(String snapshotName, final Path rootDir,
241      Configuration conf) {
242    return getSpecifiedSnapshotDir(getWorkingSnapshotDir(rootDir, conf), snapshotName);
243  }
244
245  /**
246   * Get the directory within the given filepath to store the snapshot instance
247   * @param snapshotsDir directory to store snapshot directory within
248   * @param snapshotName name of the snapshot to take
249   * @return the final directory for the snapshot in the given filepath
250   */
251  private static final Path getSpecifiedSnapshotDir(final Path snapshotsDir, String snapshotName) {
252    return new Path(snapshotsDir, snapshotName);
253  }
254
255  /**
256   * @param rootDir hbase root directory
257   * @return the directory for all completed snapshots;
258   */
259  public static final Path getSnapshotsDir(Path rootDir) {
260    return new Path(rootDir, HConstants.SNAPSHOT_DIR_NAME);
261  }
262
263  /**
264   * Determines if the given workingDir is a subdirectory of the given "root directory"
265   * @param workingDir a directory to check
266   * @param rootDir root directory of the HBase installation
267   * @return true if the given workingDir is a subdirectory of the given root directory,
268   *   false otherwise
269   */
270  public static boolean isSubDirectoryOf(final Path workingDir, final Path rootDir) {
271    return workingDir.toString().startsWith(rootDir.toString() + Path.SEPARATOR);
272  }
273
274  /**
275   * Determines if the given workingDir is a subdirectory of the default working snapshot directory
276   * @param workingDir a directory to check
277   * @param conf configuration for the HBase cluster
278   * @return true if the given workingDir is a subdirectory of the default working directory for
279   *   snapshots, false otherwise
280   * @throws IOException if we can't get the root dir
281   */
282  public static boolean isWithinDefaultWorkingDir(final Path workingDir, Configuration conf)
283    throws IOException {
284    Path defaultWorkingDir = getDefaultWorkingSnapshotDir(CommonFSUtils.getRootDir(conf));
285    return workingDir.equals(defaultWorkingDir) || isSubDirectoryOf(workingDir, defaultWorkingDir);
286  }
287
288  /**
289   * Get the default working directory for snapshots - where they are built, where they are
290   * temporarily copied on export, etc.
291   * @param rootDir root directory of the HBase installation
292   * @return Path to the default snapshot tmp directory, relative to the passed root directory
293   */
294  private static Path getDefaultWorkingSnapshotDir(final Path rootDir) {
295    return new Path(getSnapshotsDir(rootDir), SNAPSHOT_TMP_DIR_NAME);
296  }
297
298  /**
299   * Convert the passed snapshot description into a 'full' snapshot description based on default
300   * parameters, if none have been supplied. This resolves any 'optional' parameters that aren't
301   * supplied to their default values.
302   * @param snapshot general snapshot descriptor
303   * @param conf Configuration to read configured snapshot defaults if snapshot is not complete
304   * @return a valid snapshot description
305   * @throws IllegalArgumentException if the {@link SnapshotDescription} is not a complete
306   *           {@link SnapshotDescription}.
307   */
308  public static SnapshotDescription validate(SnapshotDescription snapshot, Configuration conf)
309      throws IllegalArgumentException, IOException {
310    if (!snapshot.hasTable()) {
311      throw new IllegalArgumentException(
312          "Descriptor doesn't apply to a table, so we can't build it.");
313    }
314
315    // set the creation time, if one hasn't been set
316    long time = snapshot.getCreationTime();
317    if (time == SnapshotDescriptionUtils.NO_SNAPSHOT_START_TIME_SPECIFIED) {
318      time = EnvironmentEdgeManager.currentTime();
319      LOG.debug("Creation time not specified, setting to:" + time + " (current time:"
320          + EnvironmentEdgeManager.currentTime() + ").");
321      SnapshotDescription.Builder builder = snapshot.toBuilder();
322      builder.setCreationTime(time);
323      snapshot = builder.build();
324    }
325
326    long ttl = snapshot.getTtl();
327    // set default ttl(sec) if it is not set already or the value is out of the range
328    if (ttl == SnapshotDescriptionUtils.NO_SNAPSHOT_TTL_SPECIFIED ||
329        ttl > TimeUnit.MILLISECONDS.toSeconds(Long.MAX_VALUE)) {
330      final long defaultSnapshotTtl = conf.getLong(HConstants.DEFAULT_SNAPSHOT_TTL_CONFIG_KEY,
331          HConstants.DEFAULT_SNAPSHOT_TTL);
332      if (LOG.isDebugEnabled()) {
333        LOG.debug("Snapshot current TTL value: {} resetting it to default value: {}", ttl,
334            defaultSnapshotTtl);
335      }
336      ttl = defaultSnapshotTtl;
337    }
338    SnapshotDescription.Builder builder = snapshot.toBuilder();
339    builder.setTtl(ttl);
340    snapshot = builder.build();
341
342    // set the acl to snapshot if security feature is enabled.
343    if (isSecurityAvailable(conf)) {
344      snapshot = writeAclToSnapshotDescription(snapshot, conf);
345    }
346    return snapshot;
347  }
348
349  /**
350   * Write the snapshot description into the working directory of a snapshot
351   * @param snapshot description of the snapshot being taken
352   * @param workingDir working directory of the snapshot
353   * @param fs {@link FileSystem} on which the snapshot should be taken
354   * @throws IOException if we can't reach the filesystem and the file cannot be cleaned up on
355   *           failure
356   */
357  public static void writeSnapshotInfo(SnapshotDescription snapshot, Path workingDir, FileSystem fs)
358      throws IOException {
359    FsPermission perms = CommonFSUtils.getFilePermissions(fs, fs.getConf(),
360      HConstants.DATA_FILE_UMASK_KEY);
361    Path snapshotInfo = new Path(workingDir, SnapshotDescriptionUtils.SNAPSHOTINFO_FILE);
362    try (FSDataOutputStream out = CommonFSUtils.create(fs, snapshotInfo, perms, true)){
363      snapshot.writeTo(out);
364    } catch (IOException e) {
365      // if we get an exception, try to remove the snapshot info
366      if (!fs.delete(snapshotInfo, false)) {
367        String msg = "Couldn't delete snapshot info file: " + snapshotInfo;
368        LOG.error(msg);
369        throw new IOException(msg);
370      }
371    }
372  }
373
374  /**
375   * Read in the {@link org.apache.hadoop.hbase.protobuf.generated.HBaseProtos.SnapshotDescription} stored for the snapshot in the passed directory
376   * @param fs filesystem where the snapshot was taken
377   * @param snapshotDir directory where the snapshot was stored
378   * @return the stored snapshot description
379   * @throws CorruptedSnapshotException if the
380   * snapshot cannot be read
381   */
382  public static SnapshotDescription readSnapshotInfo(FileSystem fs, Path snapshotDir)
383      throws CorruptedSnapshotException {
384    Path snapshotInfo = new Path(snapshotDir, SNAPSHOTINFO_FILE);
385    try (FSDataInputStream in = fs.open(snapshotInfo)){
386      return SnapshotDescription.parseFrom(in);
387    } catch (IOException e) {
388      throw new CorruptedSnapshotException("Couldn't read snapshot info from:" + snapshotInfo, e);
389    }
390  }
391
392  /**
393   * Commits the snapshot process by moving the working snapshot
394   * to the finalized filepath
395   *
396   * @param snapshotDir The file path of the completed snapshots
397   * @param workingDir  The file path of the in progress snapshots
398   * @param fs The file system of the completed snapshots
399   * @param workingDirFs The file system of the in progress snapshots
400   * @param conf Configuration
401   *
402   * @throws SnapshotCreationException if the snapshot could not be moved
403   * @throws IOException the filesystem could not be reached
404   */
405  public static void completeSnapshot(Path snapshotDir, Path workingDir, FileSystem fs,
406    FileSystem workingDirFs, final Configuration conf)
407    throws SnapshotCreationException, IOException {
408    LOG.debug("Sentinel is done, just moving the snapshot from " + workingDir + " to "
409      + snapshotDir);
410    // If the working and completed snapshot directory are on the same file system, attempt
411    // to rename the working snapshot directory to the completed location. If that fails,
412    // or the file systems differ, attempt to copy the directory over, throwing an exception
413    // if this fails
414    URI workingURI = workingDirFs.getUri();
415    URI rootURI = fs.getUri();
416    if ((!workingURI.getScheme().equals(rootURI.getScheme()) ||
417      workingURI.getAuthority() == null ||
418      !workingURI.getAuthority().equals(rootURI.getAuthority()) ||
419      workingURI.getUserInfo() == null ||
420      !workingURI.getUserInfo().equals(rootURI.getUserInfo()) ||
421      !fs.rename(workingDir, snapshotDir)) && !FileUtil.copy(workingDirFs, workingDir, fs,
422      snapshotDir, true, true, conf)) {
423      throw new SnapshotCreationException("Failed to copy working directory(" + workingDir
424        + ") to completed directory(" + snapshotDir + ").");
425    }
426  }
427
428  /**
429   * Check if the user is this table snapshot's owner
430   * @param snapshot the table snapshot description
431   * @param user the user
432   * @return true if the user is the owner of the snapshot,
433   *         false otherwise or the snapshot owner field is not present.
434   */
435  public static boolean isSnapshotOwner(org.apache.hadoop.hbase.client.SnapshotDescription snapshot,
436      User user) {
437    if (user == null) return false;
438    return user.getShortName().equals(snapshot.getOwner());
439  }
440
441  public static boolean isSecurityAvailable(Configuration conf) throws IOException {
442    try (Connection conn = ConnectionFactory.createConnection(conf); Admin admin = conn.getAdmin()) {
443      return admin.tableExists(PermissionStorage.ACL_TABLE_NAME);
444    }
445  }
446
447  private static SnapshotDescription writeAclToSnapshotDescription(SnapshotDescription snapshot,
448      Configuration conf) throws IOException {
449    ListMultimap<String, UserPermission> perms =
450        User.runAsLoginUser(new PrivilegedExceptionAction<ListMultimap<String, UserPermission>>() {
451          @Override
452          public ListMultimap<String, UserPermission> run() throws Exception {
453            return PermissionStorage.getTablePermissions(conf,
454              TableName.valueOf(snapshot.getTable()));
455          }
456        });
457    return snapshot.toBuilder()
458        .setUsersAndPermissions(ShadedAccessControlUtil.toUserTablePermissions(perms)).build();
459  }
460}