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.hdfs.client;
019
020import java.io.FileNotFoundException;
021import java.io.IOException;
022import java.net.URI;
023import java.util.Collection;
024import java.util.EnumSet;
025
026import org.apache.hadoop.HadoopIllegalArgumentException;
027import org.apache.hadoop.classification.InterfaceAudience;
028import org.apache.hadoop.classification.InterfaceStability;
029import org.apache.hadoop.conf.Configuration;
030import org.apache.hadoop.fs.BlockStoragePolicySpi;
031import org.apache.hadoop.fs.CacheFlag;
032import org.apache.hadoop.fs.FileEncryptionInfo;
033import org.apache.hadoop.fs.FileStatus;
034import org.apache.hadoop.fs.FileSystem;
035import org.apache.hadoop.fs.Path;
036import org.apache.hadoop.fs.RemoteIterator;
037import org.apache.hadoop.fs.StorageType;
038import org.apache.hadoop.fs.permission.FsAction;
039import org.apache.hadoop.fs.permission.FsPermission;
040import org.apache.hadoop.hdfs.DFSInotifyEventInputStream;
041import org.apache.hadoop.hdfs.DistributedFileSystem;
042import org.apache.hadoop.hdfs.protocol.CacheDirectiveEntry;
043import org.apache.hadoop.hdfs.protocol.CacheDirectiveInfo;
044import org.apache.hadoop.hdfs.protocol.CachePoolEntry;
045import org.apache.hadoop.hdfs.protocol.CachePoolInfo;
046import org.apache.hadoop.hdfs.protocol.EncryptionZone;
047import org.apache.hadoop.hdfs.protocol.HdfsConstants;
048import org.apache.hadoop.security.AccessControlException;
049
050/**
051 * The public API for performing administrative functions on HDFS. Those writing
052 * applications against HDFS should prefer this interface to directly accessing
053 * functionality in DistributedFileSystem or DFSClient.
054 *
055 * Note that this is distinct from the similarly-named DFSAdmin, which
056 * is a class that provides the functionality for the CLI `hdfs dfsadmin ...'
057 * commands.
058 */
059@InterfaceAudience.Public
060@InterfaceStability.Evolving
061public class HdfsAdmin {
062
063  private DistributedFileSystem dfs;
064  private static final FsPermission TRASH_PERMISSION = new FsPermission(
065      FsAction.ALL, FsAction.ALL, FsAction.ALL, true);
066
067  /**
068   * Create a new HdfsAdmin client.
069   *
070   * @param uri the unique URI of the HDFS file system to administer
071   * @param conf configuration
072   * @throws IOException in the event the file system could not be created
073   */
074  public HdfsAdmin(URI uri, Configuration conf) throws IOException {
075    FileSystem fs = FileSystem.get(uri, conf);
076    if (!(fs instanceof DistributedFileSystem)) {
077      throw new IllegalArgumentException("'" + uri + "' is not an HDFS URI.");
078    } else {
079      dfs = (DistributedFileSystem)fs;
080    }
081  }
082
083  /**
084   * Set the namespace quota (count of files, directories, and sym links) for a
085   * directory.
086   *
087   * @param src the path to set the quota for
088   * @param quota the value to set for the quota
089   * @throws IOException in the event of error
090   */
091  public void setQuota(Path src, long quota) throws IOException {
092    dfs.setQuota(src, quota, HdfsConstants.QUOTA_DONT_SET);
093  }
094
095  /**
096   * Clear the namespace quota (count of files, directories and sym links) for a
097   * directory.
098   *
099   * @param src the path to clear the quota of
100   * @throws IOException in the event of error
101   */
102  public void clearQuota(Path src) throws IOException {
103    dfs.setQuota(src, HdfsConstants.QUOTA_RESET, HdfsConstants.QUOTA_DONT_SET);
104  }
105
106  /**
107   * Set the storage space quota (size of files) for a directory. Note that
108   * directories and sym links do not occupy storage space.
109   *
110   * @param src the path to set the space quota of
111   * @param spaceQuota the value to set for the space quota
112   * @throws IOException in the event of error
113   */
114  public void setSpaceQuota(Path src, long spaceQuota) throws IOException {
115    dfs.setQuota(src, HdfsConstants.QUOTA_DONT_SET, spaceQuota);
116  }
117
118  /**
119   * Clear the storage space quota (size of files) for a directory. Note that
120   * directories and sym links do not occupy storage space.
121   *
122   * @param src the path to clear the space quota of
123   * @throws IOException in the event of error
124   */
125  public void clearSpaceQuota(Path src) throws IOException {
126    dfs.setQuota(src, HdfsConstants.QUOTA_DONT_SET, HdfsConstants.QUOTA_RESET);
127  }
128
129  /**
130   * Set the quota by storage type for a directory. Note that
131   * directories and sym links do not occupy storage type quota.
132   *
133   * @param src the target directory to set the quota by storage type
134   * @param type the storage type to set for quota by storage type
135   * @param quota the value to set for quota by storage type
136   * @throws IOException in the event of error
137   */
138  public void setQuotaByStorageType(Path src, StorageType type, long quota)
139      throws IOException {
140    dfs.setQuotaByStorageType(src, type, quota);
141  }
142
143  /**
144   * Clear the space quota by storage type for a directory. Note that
145   * directories and sym links do not occupy storage type quota.
146   *
147   * @param src the target directory to clear the quota by storage type
148   * @param type the storage type to clear for quota by storage type
149   * @throws IOException in the event of error
150   */
151  public void clearQuotaByStorageType(Path src, StorageType type) throws IOException {
152    dfs.setQuotaByStorageType(src, type, HdfsConstants.QUOTA_RESET);
153  }
154
155  /**
156   * Allow snapshot on a directory.
157   * @param path The path of the directory where snapshots will be taken.
158   */
159  public void allowSnapshot(Path path) throws IOException {
160    dfs.allowSnapshot(path);
161  }
162
163  /**
164   * Disallow snapshot on a directory.
165   * @param path The path of the snapshottable directory.
166   */
167  public void disallowSnapshot(Path path) throws IOException {
168    dfs.disallowSnapshot(path);
169  }
170
171  /**
172   * Add a new CacheDirectiveInfo.
173   *
174   * @param info Information about a directive to add.
175   * @param flags {@link CacheFlag}s to use for this operation.
176   * @return the ID of the directive that was created.
177   * @throws IOException if the directive could not be added
178   */
179  public long addCacheDirective(CacheDirectiveInfo info,
180      EnumSet<CacheFlag> flags) throws IOException {
181  return dfs.addCacheDirective(info, flags);
182  }
183
184  /**
185   * Modify a CacheDirective.
186   *
187   * @param info Information about the directive to modify. You must set the ID
188   *          to indicate which CacheDirective you want to modify.
189   * @param flags {@link CacheFlag}s to use for this operation.
190   * @throws IOException if the directive could not be modified
191   */
192  public void modifyCacheDirective(CacheDirectiveInfo info,
193      EnumSet<CacheFlag> flags) throws IOException {
194    dfs.modifyCacheDirective(info, flags);
195  }
196
197  /**
198   * Remove a CacheDirective.
199   *
200   * @param id identifier of the CacheDirectiveInfo to remove
201   * @throws IOException if the directive could not be removed
202   */
203  public void removeCacheDirective(long id)
204      throws IOException {
205    dfs.removeCacheDirective(id);
206  }
207
208  /**
209   * List cache directives. Incrementally fetches results from the server.
210   *
211   * @param filter Filter parameters to use when listing the directives, null to
212   *               list all directives visible to us.
213   * @return A RemoteIterator which returns CacheDirectiveInfo objects.
214   */
215  public RemoteIterator<CacheDirectiveEntry> listCacheDirectives(
216      CacheDirectiveInfo filter) throws IOException {
217    return dfs.listCacheDirectives(filter);
218  }
219
220  /**
221   * Add a cache pool.
222   *
223   * @param info
224   *          The request to add a cache pool.
225   * @throws IOException
226   *          If the request could not be completed.
227   */
228  public void addCachePool(CachePoolInfo info) throws IOException {
229    dfs.addCachePool(info);
230  }
231
232  /**
233   * Modify an existing cache pool.
234   *
235   * @param info
236   *          The request to modify a cache pool.
237   * @throws IOException
238   *          If the request could not be completed.
239   */
240  public void modifyCachePool(CachePoolInfo info) throws IOException {
241    dfs.modifyCachePool(info);
242  }
243
244  /**
245   * Remove a cache pool.
246   *
247   * @param poolName
248   *          Name of the cache pool to remove.
249   * @throws IOException
250   *          if the cache pool did not exist, or could not be removed.
251   */
252  public void removeCachePool(String poolName) throws IOException {
253    dfs.removeCachePool(poolName);
254  }
255
256  /**
257   * List all cache pools.
258   *
259   * @return A remote iterator from which you can get CachePoolEntry objects.
260   *          Requests will be made as needed.
261   * @throws IOException
262   *          If there was an error listing cache pools.
263   */
264  public RemoteIterator<CachePoolEntry> listCachePools() throws IOException {
265    return dfs.listCachePools();
266  }
267
268  /**
269   * Create an encryption zone rooted at an empty existing directory, using the
270   * specified encryption key. An encryption zone has an associated encryption
271   * key used when reading and writing files within the zone.
272   *
273   * @param path    The path of the root of the encryption zone. Must refer to
274   *                an empty, existing directory.
275   * @param keyName Name of key available at the KeyProvider.
276   * @throws IOException            if there was a general IO exception
277   * @throws AccessControlException if the caller does not have access to path
278   * @throws FileNotFoundException  if the path does not exist
279   */
280  @Deprecated
281  public void createEncryptionZone(Path path, String keyName)
282      throws IOException, AccessControlException, FileNotFoundException {
283    dfs.createEncryptionZone(path, keyName);
284  }
285
286  /**
287   * Create an encryption zone rooted at an empty existing directory, using the
288   * specified encryption key. An encryption zone has an associated encryption
289   * key used when reading and writing files within the zone.
290   *
291   * Additional options, such as provisioning the trash directory, can be
292   * specified using {@link CreateEncryptionZoneFlag} flags.
293   *
294   * @param path    The path of the root of the encryption zone. Must refer to
295   *                an empty, existing directory.
296   * @param keyName Name of key available at the KeyProvider.
297   * @param flags   flags for this operation.
298   * @throws IOException            if there was a general IO exception
299   * @throws AccessControlException if the caller does not have access to path
300   * @throws FileNotFoundException  if the path does not exist
301   * @throws HadoopIllegalArgumentException if the flags are invalid
302   */
303  public void createEncryptionZone(Path path, String keyName,
304      EnumSet<CreateEncryptionZoneFlag> flags)
305      throws IOException, AccessControlException, FileNotFoundException,
306      HadoopIllegalArgumentException{
307    dfs.createEncryptionZone(path, keyName);
308    if (flags.contains(CreateEncryptionZoneFlag.PROVISION_TRASH)) {
309      if (flags.contains(CreateEncryptionZoneFlag.NO_TRASH)) {
310        throw new HadoopIllegalArgumentException(
311            "can not have both PROVISION_TRASH and NO_TRASH flags");
312      }
313      this.provisionEZTrash(path);
314    }
315  }
316
317  /**
318   * Provision a trash directory for a given encryption zone.
319
320   * @param path the root of the encryption zone
321   * @throws IOException if the trash directory can not be created.
322   */
323  public void provisionEncryptionZoneTrash(Path path) throws IOException {
324    this.provisionEZTrash(path);
325  }
326
327  /**
328   * Get the path of the encryption zone for a given file or directory.
329   *
330   * @param path The path to get the ez for.
331   * @return An EncryptionZone, or null if path does not exist or is not in an
332   * ez.
333   * @throws IOException            if there was a general IO exception
334   * @throws AccessControlException if the caller does not have access to path
335   */
336  public EncryptionZone getEncryptionZoneForPath(Path path)
337      throws IOException, AccessControlException {
338    return dfs.getEZForPath(path);
339  }
340
341  /**
342   * Returns a RemoteIterator which can be used to list the encryption zones
343   * in HDFS. For large numbers of encryption zones, the iterator will fetch
344   * the list of zones in a number of small batches.
345   * <p/>
346   * Since the list is fetched in batches, it does not represent a
347   * consistent snapshot of the entire list of encryption zones.
348   * <p/>
349   * This method can only be called by HDFS superusers.
350   */
351  public RemoteIterator<EncryptionZone> listEncryptionZones()
352      throws IOException {
353    return dfs.listEncryptionZones();
354  }
355
356  /**
357   * Returns the FileEncryptionInfo on the HdfsFileStatus for the given path.
358   * The return value can be null if the path points to a directory, or a file
359   * that is not in an encryption zone.
360   *
361   * @throws FileNotFoundException if the path does not exist.
362   * @throws AccessControlException if no execute permission on parent path.
363   */
364  public FileEncryptionInfo getFileEncryptionInfo(final Path path)
365      throws IOException {
366    return dfs.getFileEncryptionInfo(path);
367  }
368
369  /**
370   * Exposes a stream of namesystem events. Only events occurring after the
371   * stream is created are available.
372   * See {@link org.apache.hadoop.hdfs.DFSInotifyEventInputStream}
373   * for information on stream usage.
374   * See {@link org.apache.hadoop.hdfs.inotify.Event}
375   * for information on the available events.
376   * <p/>
377   * Inotify users may want to tune the following HDFS parameters to
378   * ensure that enough extra HDFS edits are saved to support inotify clients
379   * that fall behind the current state of the namespace while reading events.
380   * The default parameter values should generally be reasonable. If edits are
381   * deleted before their corresponding events can be read, clients will see a
382   * {@link org.apache.hadoop.hdfs.inotify.MissingEventsException} on
383   * {@link org.apache.hadoop.hdfs.DFSInotifyEventInputStream} method calls.
384   *
385   * It should generally be sufficient to tune these parameters:
386   * dfs.namenode.num.extra.edits.retained
387   * dfs.namenode.max.extra.edits.segments.retained
388   *
389   * Parameters that affect the number of created segments and the number of
390   * edits that are considered necessary, i.e. do not count towards the
391   * dfs.namenode.num.extra.edits.retained quota):
392   * dfs.namenode.checkpoint.period
393   * dfs.namenode.checkpoint.txns
394   * dfs.namenode.num.checkpoints.retained
395   * dfs.ha.log-roll.period
396   * <p/>
397   * It is recommended that local journaling be configured
398   * (dfs.namenode.edits.dir) for inotify (in addition to a shared journal)
399   * so that edit transfers from the shared journal can be avoided.
400   *
401   * @throws IOException If there was an error obtaining the stream.
402   */
403  public DFSInotifyEventInputStream getInotifyEventStream() throws IOException {
404    return dfs.getInotifyEventStream();
405  }
406
407  /**
408   * A version of {@link HdfsAdmin#getInotifyEventStream()} meant for advanced
409   * users who are aware of HDFS edits up to lastReadTxid (e.g. because they
410   * have access to an FSImage inclusive of lastReadTxid) and only want to read
411   * events after this point.
412   */
413  public DFSInotifyEventInputStream getInotifyEventStream(long lastReadTxid)
414      throws IOException {
415    return dfs.getInotifyEventStream(lastReadTxid);
416  }
417
418  /**
419   * Set the source path to the specified storage policy.
420   *
421   * @param src The source path referring to either a directory or a file.
422   * @param policyName The name of the storage policy.
423   */
424  public void setStoragePolicy(final Path src, final String policyName)
425      throws IOException {
426    dfs.setStoragePolicy(src, policyName);
427  }
428
429  /**
430   * Unset the storage policy set for a given file or directory.
431   *
432   * @param src file or directory path.
433   * @throws IOException
434   */
435  public void unsetStoragePolicy(final Path src) throws IOException {
436    dfs.unsetStoragePolicy(src);
437  }
438
439  /**
440   * Query the effective storage policy ID for the given file or directory.
441   *
442   * @param src file or directory path.
443   * @return storage policy for the given file or directory.
444   * @throws IOException
445   */
446  public BlockStoragePolicySpi getStoragePolicy(final Path src)
447      throws IOException {
448    return dfs.getStoragePolicy(src);
449  }
450
451  /**
452   * Retrieve all the storage policies supported by HDFS file system.
453   *
454   * @return all storage policies supported by HDFS file system.
455   * @throws IOException
456   */
457  public Collection<? extends BlockStoragePolicySpi> getAllStoragePolicies()
458      throws IOException {
459    return dfs.getAllStoragePolicies();
460  }
461
462  private void provisionEZTrash(Path path) throws IOException {
463    // make sure the path is an EZ
464    EncryptionZone ez = dfs.getEZForPath(path);
465    if (ez == null) {
466      throw new IllegalArgumentException(path + " is not an encryption zone.");
467    }
468
469    String ezPath = ez.getPath();
470    if (!path.toString().equals(ezPath)) {
471      throw new IllegalArgumentException(path + " is not the root of an " +
472          "encryption zone. Do you mean " + ez.getPath() + "?");
473    }
474
475    // check if the trash directory exists
476
477    Path trashPath = new Path(ez.getPath(), FileSystem.TRASH_PREFIX);
478
479    if (dfs.exists(trashPath)) {
480      String errMessage = "Will not provision new trash directory for " +
481          "encryption zone " + ez.getPath() + ". Path already exists.";
482      FileStatus trashFileStatus = dfs.getFileStatus(trashPath);
483      if (!trashFileStatus.isDirectory()) {
484        errMessage += "\r\n" +
485            "Warning: " + trashPath.toString() + " is not a directory";
486      }
487      if (!trashFileStatus.getPermission().equals(TRASH_PERMISSION)) {
488        errMessage += "\r\n" +
489            "Warning: the permission of " +
490            trashPath.toString() + " is not " + TRASH_PERMISSION;
491      }
492      throw new IOException(errMessage);
493    }
494
495    // Update the permission bits
496    dfs.mkdir(trashPath, TRASH_PERMISSION);
497    dfs.setPermission(trashPath, TRASH_PERMISSION);
498  }
499}