001/**
002 *
003 * Licensed to the Apache Software Foundation (ASF) under one
004 * or more contributor license agreements.  See the NOTICE file
005 * distributed with this work for additional information
006 * regarding copyright ownership.  The ASF licenses this file
007 * to you under the Apache License, Version 2.0 (the
008 * "License"); you may not use this file except in compliance
009 * with the License.  You may obtain a copy of the License at
010 *
011 *     http://www.apache.org/licenses/LICENSE-2.0
012 *
013 * Unless required by applicable law or agreed to in writing, software
014 * distributed under the License is distributed on an "AS IS" BASIS,
015 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
016 * See the License for the specific language governing permissions and
017 * limitations under the License.
018 */
019package org.apache.hadoop.hbase.master;
020
021import java.io.FileNotFoundException;
022import java.io.IOException;
023import org.apache.hadoop.conf.Configuration;
024import org.apache.hadoop.fs.FileStatus;
025import org.apache.hadoop.fs.FileSystem;
026import org.apache.hadoop.fs.Path;
027import org.apache.hadoop.fs.permission.FsAction;
028import org.apache.hadoop.fs.permission.FsPermission;
029import org.apache.hadoop.hbase.ClusterId;
030import org.apache.hadoop.hbase.HConstants;
031import org.apache.hadoop.hbase.backup.HFileArchiver;
032import org.apache.hadoop.hbase.client.RegionInfo;
033import org.apache.hadoop.hbase.exceptions.DeserializationException;
034import org.apache.hadoop.hbase.fs.HFileSystem;
035import org.apache.hadoop.hbase.log.HBaseMarkers;
036import org.apache.hadoop.hbase.mob.MobConstants;
037import org.apache.hadoop.hbase.security.access.SnapshotScannerHDFSAclHelper;
038import org.apache.hadoop.hbase.util.Bytes;
039import org.apache.hadoop.hbase.util.CommonFSUtils;
040import org.apache.hadoop.hbase.util.FSUtils;
041import org.apache.yetus.audience.InterfaceAudience;
042import org.slf4j.Logger;
043import org.slf4j.LoggerFactory;
044
045/**
046 * This class abstracts a bunch of operations the HMaster needs to interact with
047 * the underlying file system like creating the initial layout, checking file
048 * system status, etc.
049 */
050@InterfaceAudience.Private
051public class MasterFileSystem {
052  private static final Logger LOG = LoggerFactory.getLogger(MasterFileSystem.class);
053
054  /** Parameter name for HBase instance root directory permission*/
055  public static final String HBASE_DIR_PERMS = "hbase.rootdir.perms";
056
057  /** Parameter name for HBase WAL directory permission*/
058  public static final String HBASE_WAL_DIR_PERMS = "hbase.wal.dir.perms";
059
060  // HBase configuration
061  private final Configuration conf;
062  // Persisted unique cluster ID
063  private ClusterId clusterId;
064  // Keep around for convenience.
065  private final FileSystem fs;
066  // Keep around for convenience.
067  private final FileSystem walFs;
068  // root log directory on the FS
069  private final Path rootdir;
070  // hbase temp directory used for table construction and deletion
071  private final Path tempdir;
072  // root hbase directory on the FS
073  private final Path walRootDir;
074
075
076  /*
077   * In a secure env, the protected sub-directories and files under the HBase rootDir
078   * would be restricted. The sub-directory will have '700' except the bulk load staging dir,
079   * which will have '711'.  The default '700' can be overwritten by setting the property
080   * 'hbase.rootdir.perms'. The protected files (version file, clusterId file) will have '600'.
081   * The rootDir itself will be created with HDFS default permissions if it does not exist.
082   * We will check the rootDir permissions to make sure it has 'x' for all to ensure access
083   * to the staging dir. If it does not, we will add it.
084   */
085  // Permissions for the directories under rootDir that need protection
086  private final FsPermission secureRootSubDirPerms;
087  // Permissions for the files under rootDir that need protection
088  private final FsPermission secureRootFilePerms = new FsPermission("600");
089  // Permissions for bulk load staging directory under rootDir
090  private final FsPermission HiddenDirPerms = FsPermission.valueOf("-rwx--x--x");
091
092  private boolean isSecurityEnabled;
093
094  public MasterFileSystem(Configuration conf) throws IOException {
095    this.conf = conf;
096    // Set filesystem to be that of this.rootdir else we get complaints about
097    // mismatched filesystems if hbase.rootdir is hdfs and fs.defaultFS is
098    // default localfs.  Presumption is that rootdir is fully-qualified before
099    // we get to here with appropriate fs scheme.
100    this.rootdir = CommonFSUtils.getRootDir(conf);
101    this.tempdir = new Path(this.rootdir, HConstants.HBASE_TEMP_DIRECTORY);
102    // Cover both bases, the old way of setting default fs and the new.
103    // We're supposed to run on 0.20 and 0.21 anyways.
104    this.fs = this.rootdir.getFileSystem(conf);
105    this.walRootDir = CommonFSUtils.getWALRootDir(conf);
106    this.walFs = CommonFSUtils.getWALFileSystem(conf);
107    CommonFSUtils.setFsDefault(conf, new Path(this.walFs.getUri()));
108    walFs.setConf(conf);
109    CommonFSUtils.setFsDefault(conf, new Path(this.fs.getUri()));
110    // make sure the fs has the same conf
111    fs.setConf(conf);
112    this.secureRootSubDirPerms = new FsPermission(conf.get("hbase.rootdir.perms", "700"));
113    this.isSecurityEnabled = "kerberos".equalsIgnoreCase(conf.get("hbase.security.authentication"));
114    // setup the filesystem variable
115    createInitialFileSystemLayout();
116    HFileSystem.addLocationsOrderInterceptor(conf);
117  }
118
119  /**
120   * Create initial layout in filesystem.
121   * <ol>
122   * <li>Check if the meta region exists and is readable, if not create it.
123   * Create hbase.version and the hbase:meta directory if not one.
124   * </li>
125   * </ol>
126   * Idempotent.
127   */
128  private void createInitialFileSystemLayout() throws IOException {
129
130    final String[] protectedSubDirs = new String[] {
131        HConstants.BASE_NAMESPACE_DIR,
132        HConstants.HFILE_ARCHIVE_DIRECTORY,
133        HConstants.HBCK_SIDELINEDIR_NAME,
134        MobConstants.MOB_DIR_NAME
135    };
136
137    //With the introduction of RegionProcedureStore,
138    // there's no need to create MasterProcWAL dir here anymore. See HBASE-23715
139    final String[] protectedSubLogDirs = new String[] {
140      HConstants.HREGION_LOGDIR_NAME,
141      HConstants.HREGION_OLDLOGDIR_NAME,
142      HConstants.CORRUPT_DIR_NAME
143    };
144    // check if the root directory exists
145    checkRootDir(this.rootdir, conf, this.fs);
146
147    // Check the directories under rootdir.
148    checkTempDir(this.tempdir, conf, this.fs);
149    for (String subDir : protectedSubDirs) {
150      checkSubDir(new Path(this.rootdir, subDir), HBASE_DIR_PERMS);
151    }
152
153    final String perms;
154    if (!this.walRootDir.equals(this.rootdir)) {
155      perms = HBASE_WAL_DIR_PERMS;
156    } else {
157      perms = HBASE_DIR_PERMS;
158    }
159    for (String subDir : protectedSubLogDirs) {
160      checkSubDir(new Path(this.walRootDir, subDir), perms);
161    }
162
163    checkStagingDir();
164
165    // Handle the last few special files and set the final rootDir permissions
166    // rootDir needs 'x' for all to support bulk load staging dir
167    if (isSecurityEnabled) {
168      fs.setPermission(new Path(rootdir, HConstants.VERSION_FILE_NAME), secureRootFilePerms);
169      fs.setPermission(new Path(rootdir, HConstants.CLUSTER_ID_FILE_NAME), secureRootFilePerms);
170    }
171    FsPermission currentRootPerms = fs.getFileStatus(this.rootdir).getPermission();
172    if (!currentRootPerms.getUserAction().implies(FsAction.EXECUTE)
173        || !currentRootPerms.getGroupAction().implies(FsAction.EXECUTE)
174        || !currentRootPerms.getOtherAction().implies(FsAction.EXECUTE)) {
175      LOG.warn("rootdir permissions do not contain 'excute' for user, group or other. "
176        + "Automatically adding 'excute' permission for all");
177      fs.setPermission(
178        this.rootdir,
179        new FsPermission(currentRootPerms.getUserAction().or(FsAction.EXECUTE), currentRootPerms
180            .getGroupAction().or(FsAction.EXECUTE), currentRootPerms.getOtherAction().or(
181          FsAction.EXECUTE)));
182    }
183  }
184
185  public FileSystem getFileSystem() {
186    return this.fs;
187  }
188
189  public FileSystem getWALFileSystem() {
190    return this.walFs;
191  }
192
193  public Configuration getConfiguration() {
194    return this.conf;
195  }
196
197  /**
198   * @return HBase root dir.
199   */
200  public Path getRootDir() {
201    return this.rootdir;
202  }
203
204  /**
205   * @return HBase root log dir.
206   */
207  public Path getWALRootDir() {
208    return this.walRootDir;
209  }
210
211  /**
212   * @return the directory for a give {@code region}.
213   */
214  public Path getRegionDir(RegionInfo region) {
215    return FSUtils.getRegionDirFromRootDir(getRootDir(), region);
216  }
217
218  /**
219   * @return HBase temp dir.
220   */
221  public Path getTempDir() {
222    return this.tempdir;
223  }
224
225  /**
226   * @return The unique identifier generated for this cluster
227   */
228  public ClusterId getClusterId() {
229    return clusterId;
230  }
231
232  /**
233   * Get the rootdir.  Make sure its wholesome and exists before returning.
234   * @param rd
235   * @param c
236   * @param fs
237   * @return hbase.rootdir (after checks for existence and bootstrapping if
238   * needed populating the directory with necessary bootup files).
239   * @throws IOException
240   */
241  private void checkRootDir(final Path rd, final Configuration c, final FileSystem fs)
242    throws IOException {
243    int threadWakeFrequency = c.getInt(HConstants.THREAD_WAKE_FREQUENCY, 10 * 1000);
244    // If FS is in safe mode wait till out of it.
245    FSUtils.waitOnSafeMode(c, threadWakeFrequency);
246
247    // Filesystem is good. Go ahead and check for hbase.rootdir.
248    FileStatus status;
249    try {
250      status = fs.getFileStatus(rd);
251    } catch (FileNotFoundException e) {
252      status = null;
253    }
254    int versionFileWriteAttempts = c.getInt(HConstants.VERSION_FILE_WRITE_ATTEMPTS,
255      HConstants.DEFAULT_VERSION_FILE_WRITE_ATTEMPTS);
256    try {
257      if (status == null) {
258        if (!fs.mkdirs(rd)) {
259          throw new IOException("Can not create configured '" + HConstants.HBASE_DIR + "' " + rd);
260        }
261        // DFS leaves safe mode with 0 DNs when there are 0 blocks.
262        // We used to handle this by checking the current DN count and waiting until
263        // it is nonzero. With security, the check for datanode count doesn't work --
264        // it is a privileged op. So instead we adopt the strategy of the jobtracker
265        // and simply retry file creation during bootstrap indefinitely. As soon as
266        // there is one datanode it will succeed. Permission problems should have
267        // already been caught by mkdirs above.
268        FSUtils.setVersion(fs, rd, threadWakeFrequency, versionFileWriteAttempts);
269      } else {
270        if (!status.isDirectory()) {
271          throw new IllegalArgumentException(
272            "Configured '" + HConstants.HBASE_DIR + "' " + rd + " is not a directory.");
273        }
274        // as above
275        FSUtils.checkVersion(fs, rd, true, threadWakeFrequency, versionFileWriteAttempts);
276      }
277    } catch (DeserializationException de) {
278      LOG.error(HBaseMarkers.FATAL, "Please fix invalid configuration for '{}' {}",
279        HConstants.HBASE_DIR, rd, de);
280      throw new IOException(de);
281    } catch (IllegalArgumentException iae) {
282      LOG.error(HBaseMarkers.FATAL, "Please fix invalid configuration for '{}' {}",
283        HConstants.HBASE_DIR, rd, iae);
284      throw iae;
285    }
286    // Make sure cluster ID exists
287    if (!FSUtils.checkClusterIdExists(fs, rd, threadWakeFrequency)) {
288      FSUtils.setClusterId(fs, rd, new ClusterId(), threadWakeFrequency);
289    }
290    clusterId = FSUtils.getClusterId(fs, rd);
291  }
292
293  /**
294   * Make sure the hbase temp directory exists and is empty.
295   * NOTE that this method is only executed once just after the master becomes the active one.
296   */
297  void checkTempDir(final Path tmpdir, final Configuration c, final FileSystem fs)
298      throws IOException {
299    // If the temp directory exists, clear the content (left over, from the previous run)
300    if (fs.exists(tmpdir)) {
301      // Archive table in temp, maybe left over from failed deletion,
302      // if not the cleaner will take care of them.
303      for (Path tableDir: FSUtils.getTableDirs(fs, tmpdir)) {
304        HFileArchiver.archiveRegions(c, fs, this.rootdir, tableDir,
305          FSUtils.getRegionDirs(fs, tableDir));
306        if (!FSUtils.getRegionDirs(fs, tableDir).isEmpty()) {
307          LOG.warn("Found regions in tmp dir after archiving table regions, {}", tableDir);
308        }
309      }
310      // if acl sync to hdfs is enabled, then skip delete tmp dir because ACLs are set
311      if (!SnapshotScannerHDFSAclHelper.isAclSyncToHdfsEnabled(c) && !fs.delete(tmpdir, true)) {
312        throw new IOException("Unable to clean the temp directory: " + tmpdir);
313      }
314    }
315
316    // Create the temp directory
317    if (!fs.exists(tmpdir)) {
318      if (isSecurityEnabled) {
319        if (!fs.mkdirs(tmpdir, secureRootSubDirPerms)) {
320          throw new IOException("HBase temp directory '" + tmpdir + "' creation failure.");
321        }
322      } else {
323        if (!fs.mkdirs(tmpdir)) {
324          throw new IOException("HBase temp directory '" + tmpdir + "' creation failure.");
325        }
326      }
327    }
328  }
329
330  /**
331   * Make sure the directories under rootDir have good permissions. Create if necessary.
332   * @param p
333   * @throws IOException
334   */
335  private void checkSubDir(final Path p, final String dirPermsConfName) throws IOException {
336    FileSystem fs = p.getFileSystem(conf);
337    FsPermission dirPerms = new FsPermission(conf.get(dirPermsConfName, "700"));
338    if (!fs.exists(p)) {
339      if (isSecurityEnabled) {
340        if (!fs.mkdirs(p, secureRootSubDirPerms)) {
341          throw new IOException("HBase directory '" + p + "' creation failure.");
342        }
343      } else {
344        if (!fs.mkdirs(p)) {
345          throw new IOException("HBase directory '" + p + "' creation failure.");
346        }
347      }
348    }
349    else {
350      if (isSecurityEnabled && !dirPerms.equals(fs.getFileStatus(p).getPermission())) {
351        // check whether the permission match
352        LOG.warn("Found HBase directory permissions NOT matching expected permissions for "
353            + p.toString() + " permissions=" + fs.getFileStatus(p).getPermission()
354            + ", expecting " + dirPerms + ". Automatically setting the permissions. "
355            + "You can change the permissions by setting \"" + dirPermsConfName + "\" in hbase-site.xml "
356            + "and restarting the master");
357        fs.setPermission(p, dirPerms);
358      }
359    }
360  }
361
362  /**
363   * Check permissions for bulk load staging directory. This directory has special hidden
364   * permissions. Create it if necessary.
365   * @throws IOException
366   */
367  private void checkStagingDir() throws IOException {
368    Path p = new Path(this.rootdir, HConstants.BULKLOAD_STAGING_DIR_NAME);
369    try {
370      if (!this.fs.exists(p)) {
371        if (!this.fs.mkdirs(p, HiddenDirPerms)) {
372          throw new IOException("Failed to create staging directory " + p.toString());
373        }
374      } else {
375        this.fs.setPermission(p, HiddenDirPerms);
376      }
377    } catch (IOException e) {
378      LOG.error("Failed to create or set permission on staging directory " + p.toString());
379      throw new IOException("Failed to create or set permission on staging directory "
380          + p.toString(), e);
381    }
382  }
383
384  public void deleteFamilyFromFS(RegionInfo region, byte[] familyName)
385      throws IOException {
386    deleteFamilyFromFS(rootdir, region, familyName);
387  }
388
389  public void deleteFamilyFromFS(Path rootDir, RegionInfo region, byte[] familyName)
390      throws IOException {
391    // archive family store files
392    Path tableDir = CommonFSUtils.getTableDir(rootDir, region.getTable());
393    HFileArchiver.archiveFamily(fs, conf, region, tableDir, familyName);
394
395    // delete the family folder
396    Path familyDir = new Path(tableDir,
397      new Path(region.getEncodedName(), Bytes.toString(familyName)));
398    if (fs.delete(familyDir, true) == false) {
399      if (fs.exists(familyDir)) {
400        throw new IOException("Could not delete family "
401            + Bytes.toString(familyName) + " from FileSystem for region "
402            + region.getRegionNameAsString() + "(" + region.getEncodedName()
403            + ")");
404      }
405    }
406  }
407
408  public void stop() {
409  }
410
411  public void logFileSystemState(Logger log) throws IOException {
412    CommonFSUtils.logFileSystemState(fs, rootdir, log);
413  }
414}