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.server.namenode;
019
020import java.io.FileNotFoundException;
021import java.io.IOException;
022import java.util.EnumSet;
023import java.util.List;
024import java.util.Map;
025import java.util.NavigableMap;
026import java.util.TreeMap;
027
028import com.google.common.base.Preconditions;
029import com.google.common.collect.Lists;
030import org.apache.hadoop.conf.Configuration;
031import org.apache.hadoop.crypto.CipherSuite;
032import org.apache.hadoop.crypto.CryptoProtocolVersion;
033import org.apache.hadoop.fs.UnresolvedLinkException;
034import org.apache.hadoop.fs.XAttr;
035import org.apache.hadoop.fs.XAttrSetFlag;
036import org.apache.hadoop.hdfs.DFSConfigKeys;
037import org.apache.hadoop.hdfs.XAttrHelper;
038import org.apache.hadoop.hdfs.protocol.EncryptionZone;
039import org.apache.hadoop.hdfs.protocol.SnapshotAccessControlException;
040import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos;
041import org.apache.hadoop.hdfs.protocolPB.PBHelperClient;
042import org.apache.hadoop.hdfs.server.namenode.FSDirectory.DirOp;
043import org.slf4j.Logger;
044import org.slf4j.LoggerFactory;
045
046
047import static org.apache.hadoop.fs.BatchedRemoteIterator.BatchedListEntries;
048import static org.apache.hadoop.hdfs.server.common.HdfsServerConstants
049    .CRYPTO_XATTR_ENCRYPTION_ZONE;
050
051/**
052 * Manages the list of encryption zones in the filesystem.
053 * <p/>
054 * The EncryptionZoneManager has its own lock, but relies on the FSDirectory
055 * lock being held for many operations. The FSDirectory lock should not be
056 * taken if the manager lock is already held.
057 */
058public class EncryptionZoneManager {
059
060  public static Logger LOG = LoggerFactory.getLogger(EncryptionZoneManager
061      .class);
062
063  /**
064   * EncryptionZoneInt is the internal representation of an encryption zone. The
065   * external representation of an EZ is embodied in an EncryptionZone and
066   * contains the EZ's pathname.
067   */
068  private static class EncryptionZoneInt {
069    private final long inodeId;
070    private final CipherSuite suite;
071    private final CryptoProtocolVersion version;
072    private final String keyName;
073
074    EncryptionZoneInt(long inodeId, CipherSuite suite,
075        CryptoProtocolVersion version, String keyName) {
076      Preconditions.checkArgument(suite != CipherSuite.UNKNOWN);
077      Preconditions.checkArgument(version != CryptoProtocolVersion.UNKNOWN);
078      this.inodeId = inodeId;
079      this.suite = suite;
080      this.version = version;
081      this.keyName = keyName;
082    }
083
084    long getINodeId() {
085      return inodeId;
086    }
087
088    CipherSuite getSuite() {
089      return suite;
090    }
091
092    CryptoProtocolVersion getVersion() { return version; }
093
094    String getKeyName() {
095      return keyName;
096    }
097  }
098
099  private TreeMap<Long, EncryptionZoneInt> encryptionZones = null;
100  private final FSDirectory dir;
101  private final int maxListEncryptionZonesResponses;
102
103  /**
104   * Construct a new EncryptionZoneManager.
105   *
106   * @param dir Enclosing FSDirectory
107   */
108  public EncryptionZoneManager(FSDirectory dir, Configuration conf) {
109    this.dir = dir;
110    maxListEncryptionZonesResponses = conf.getInt(
111        DFSConfigKeys.DFS_NAMENODE_LIST_ENCRYPTION_ZONES_NUM_RESPONSES,
112        DFSConfigKeys.DFS_NAMENODE_LIST_ENCRYPTION_ZONES_NUM_RESPONSES_DEFAULT
113    );
114    Preconditions.checkArgument(maxListEncryptionZonesResponses >= 0,
115        DFSConfigKeys.DFS_NAMENODE_LIST_ENCRYPTION_ZONES_NUM_RESPONSES + " " +
116            "must be a positive integer."
117    );
118  }
119
120  /**
121   * Add a new encryption zone.
122   * <p/>
123   * Called while holding the FSDirectory lock.
124   *
125   * @param inodeId of the encryption zone
126   * @param keyName encryption zone key name
127   */
128  void addEncryptionZone(Long inodeId, CipherSuite suite,
129      CryptoProtocolVersion version, String keyName) {
130    assert dir.hasWriteLock();
131    unprotectedAddEncryptionZone(inodeId, suite, version, keyName);
132  }
133
134  /**
135   * Add a new encryption zone.
136   * <p/>
137   * Does not assume that the FSDirectory lock is held.
138   *
139   * @param inodeId of the encryption zone
140   * @param keyName encryption zone key name
141   */
142  void unprotectedAddEncryptionZone(Long inodeId,
143      CipherSuite suite, CryptoProtocolVersion version, String keyName) {
144    final EncryptionZoneInt ez = new EncryptionZoneInt(
145        inodeId, suite, version, keyName);
146    if (encryptionZones == null) {
147      encryptionZones = new TreeMap<>();
148    }
149    encryptionZones.put(inodeId, ez);
150  }
151
152  /**
153   * Remove an encryption zone.
154   * <p/>
155   * Called while holding the FSDirectory lock.
156   */
157  void removeEncryptionZone(Long inodeId) {
158    assert dir.hasWriteLock();
159    if (hasCreatedEncryptionZone()) {
160      encryptionZones.remove(inodeId);
161    }
162  }
163
164  /**
165   * Returns true if an IIP is within an encryption zone.
166   * <p/>
167   * Called while holding the FSDirectory lock.
168   */
169  boolean isInAnEZ(INodesInPath iip)
170      throws UnresolvedLinkException, SnapshotAccessControlException {
171    assert dir.hasReadLock();
172    return (getEncryptionZoneForPath(iip) != null);
173  }
174
175  /**
176   * Returns the path of the EncryptionZoneInt.
177   * <p/>
178   * Called while holding the FSDirectory lock.
179   */
180  private String getFullPathName(EncryptionZoneInt ezi) {
181    assert dir.hasReadLock();
182    return dir.getInode(ezi.getINodeId()).getFullPathName();
183  }
184
185  /**
186   * Get the key name for an encryption zone. Returns null if <tt>iip</tt> is
187   * not within an encryption zone.
188   * <p/>
189   * Called while holding the FSDirectory lock.
190   */
191  String getKeyName(final INodesInPath iip) {
192    assert dir.hasReadLock();
193    EncryptionZoneInt ezi = getEncryptionZoneForPath(iip);
194    if (ezi == null) {
195      return null;
196    }
197    return ezi.getKeyName();
198  }
199
200  /**
201   * Looks up the EncryptionZoneInt for a path within an encryption zone.
202   * Returns null if path is not within an EZ.
203   * <p/>
204   * Called while holding the FSDirectory lock.
205   */
206  private EncryptionZoneInt getEncryptionZoneForPath(INodesInPath iip) {
207    assert dir.hasReadLock();
208    Preconditions.checkNotNull(iip);
209    if (!hasCreatedEncryptionZone()) {
210      return null;
211    }
212    List<INode> inodes = iip.getReadOnlyINodes();
213    for (int i = inodes.size() - 1; i >= 0; i--) {
214      final INode inode = inodes.get(i);
215      if (inode != null) {
216        final EncryptionZoneInt ezi = encryptionZones.get(inode.getId());
217        if (ezi != null) {
218          return ezi;
219        }
220      }
221    }
222    return null;
223  }
224
225  /**
226   * Looks up the nearest ancestor EncryptionZoneInt that contains the given
227   * path (excluding itself).
228   * Returns null if path is not within an EZ, or the path is the root dir '/'
229   * <p/>
230   * Called while holding the FSDirectory lock.
231   */
232  private EncryptionZoneInt getParentEncryptionZoneForPath(INodesInPath iip) {
233    assert dir.hasReadLock();
234    Preconditions.checkNotNull(iip);
235    INodesInPath parentIIP = iip.getParentINodesInPath();
236    return parentIIP == null ? null : getEncryptionZoneForPath(parentIIP);
237  }
238
239  /**
240   * Returns an EncryptionZone representing the ez for a given path.
241   * Returns an empty marker EncryptionZone if path is not in an ez.
242   *
243   * @param iip The INodesInPath of the path to check
244   * @return the EncryptionZone representing the ez for the path.
245   */
246  EncryptionZone getEZINodeForPath(INodesInPath iip) {
247    final EncryptionZoneInt ezi = getEncryptionZoneForPath(iip);
248    if (ezi == null) {
249      return null;
250    } else {
251      return new EncryptionZone(ezi.getINodeId(), getFullPathName(ezi),
252          ezi.getSuite(), ezi.getVersion(), ezi.getKeyName());
253    }
254  }
255
256  /**
257   * Throws an exception if the provided path cannot be renamed into the
258   * destination because of differing parent encryption zones.
259   * <p/>
260   * Called while holding the FSDirectory lock.
261   *
262   * @param srcIIP source IIP
263   * @param dstIIP destination IIP
264   * @throws IOException if the src cannot be renamed to the dst
265   */
266  void checkMoveValidity(INodesInPath srcIIP, INodesInPath dstIIP)
267      throws IOException {
268    assert dir.hasReadLock();
269    if (!hasCreatedEncryptionZone()) {
270      return;
271    }
272    final EncryptionZoneInt srcParentEZI =
273        getParentEncryptionZoneForPath(srcIIP);
274    final EncryptionZoneInt dstParentEZI =
275        getParentEncryptionZoneForPath(dstIIP);
276    final boolean srcInEZ = (srcParentEZI != null);
277    final boolean dstInEZ = (dstParentEZI != null);
278    if (srcInEZ && !dstInEZ) {
279      throw new IOException(
280          srcIIP.getPath() + " can't be moved from an encryption zone.");
281    } else if (dstInEZ && !srcInEZ) {
282      throw new IOException(
283          srcIIP.getPath() + " can't be moved into an encryption zone.");
284    }
285
286    if (srcInEZ) {
287      if (srcParentEZI != dstParentEZI) {
288        final String srcEZPath = getFullPathName(srcParentEZI);
289        final String dstEZPath = getFullPathName(dstParentEZI);
290        final StringBuilder sb = new StringBuilder(srcIIP.getPath());
291        sb.append(" can't be moved from encryption zone ");
292        sb.append(srcEZPath);
293        sb.append(" to encryption zone ");
294        sb.append(dstEZPath);
295        sb.append(".");
296        throw new IOException(sb.toString());
297      }
298    }
299  }
300
301  /**
302   * Create a new encryption zone.
303   * <p/>
304   * Called while holding the FSDirectory lock.
305   */
306  XAttr createEncryptionZone(INodesInPath srcIIP, CipherSuite suite,
307      CryptoProtocolVersion version, String keyName)
308      throws IOException {
309    assert dir.hasWriteLock();
310
311    // Check if src is a valid path for new EZ creation
312    if (srcIIP.getLastINode() == null) {
313      throw new FileNotFoundException("cannot find " + srcIIP.getPath());
314    }
315    if (dir.isNonEmptyDirectory(srcIIP)) {
316      throw new IOException(
317          "Attempt to create an encryption zone for a non-empty directory.");
318    }
319
320    INode srcINode = srcIIP.getLastINode();
321    if (!srcINode.isDirectory()) {
322      throw new IOException("Attempt to create an encryption zone for a file.");
323    }
324
325    if (hasCreatedEncryptionZone() && encryptionZones.
326        get(srcINode.getId()) != null) {
327      throw new IOException(
328          "Directory " + srcIIP.getPath() + " is already an encryption zone.");
329    }
330
331    final HdfsProtos.ZoneEncryptionInfoProto proto =
332        PBHelperClient.convert(suite, version, keyName);
333    final XAttr ezXAttr = XAttrHelper
334        .buildXAttr(CRYPTO_XATTR_ENCRYPTION_ZONE, proto.toByteArray());
335
336    final List<XAttr> xattrs = Lists.newArrayListWithCapacity(1);
337    xattrs.add(ezXAttr);
338    // updating the xattr will call addEncryptionZone,
339    // done this way to handle edit log loading
340    FSDirXAttrOp.unprotectedSetXAttrs(dir, srcIIP, xattrs,
341                                      EnumSet.of(XAttrSetFlag.CREATE));
342    return ezXAttr;
343  }
344
345  /**
346   * Cursor-based listing of encryption zones.
347   * <p/>
348   * Called while holding the FSDirectory lock.
349   */
350  BatchedListEntries<EncryptionZone> listEncryptionZones(long prevId)
351      throws IOException {
352    assert dir.hasReadLock();
353    if (!hasCreatedEncryptionZone()) {
354      final List<EncryptionZone> emptyZones = Lists.newArrayList();
355      return new BatchedListEntries<EncryptionZone>(emptyZones, false);
356    }
357    NavigableMap<Long, EncryptionZoneInt> tailMap = encryptionZones.tailMap
358        (prevId, false);
359    final int numResponses = Math.min(maxListEncryptionZonesResponses,
360        tailMap.size());
361    final List<EncryptionZone> zones =
362        Lists.newArrayListWithExpectedSize(numResponses);
363
364    int count = 0;
365    for (EncryptionZoneInt ezi : tailMap.values()) {
366      /*
367       Skip EZs that are only present in snapshots. Re-resolve the path to 
368       see if the path's current inode ID matches EZ map's INode ID.
369       
370       INode#getFullPathName simply calls getParent recursively, so will return
371       the INode's parents at the time it was snapshotted. It will not 
372       contain a reference INode.
373      */
374      final String pathName = getFullPathName(ezi);
375      INode inode = dir.getInode(ezi.getINodeId());
376      INode lastINode = null;
377      if (inode.getParent() != null || inode.isRoot()) {
378        INodesInPath iip = dir.getINodesInPath(pathName, DirOp.READ_LINK);
379        lastINode = iip.getLastINode();
380      }
381      if (lastINode == null || lastINode.getId() != ezi.getINodeId()) {
382        continue;
383      }
384      // Add the EZ to the result list
385      zones.add(new EncryptionZone(ezi.getINodeId(), pathName,
386          ezi.getSuite(), ezi.getVersion(), ezi.getKeyName()));
387      count++;
388      if (count >= numResponses) {
389        break;
390      }
391    }
392    final boolean hasMore = (numResponses < tailMap.size());
393    return new BatchedListEntries<EncryptionZone>(zones, hasMore);
394  }
395
396  /**
397   * @return number of encryption zones.
398   */
399  public int getNumEncryptionZones() {
400    return hasCreatedEncryptionZone() ?
401        encryptionZones.size() : 0;
402  }
403
404  /**
405   * @return Whether there has been any attempt to create an encryption zone in
406   * the cluster at all. If not, it is safe to quickly return null when
407   * checking the encryption information of any file or directory in the
408   * cluster.
409   */
410  public boolean hasCreatedEncryptionZone() {
411    return encryptionZones != null;
412  }
413
414  /**
415   * @return a list of all key names.
416   */
417  String[] getKeyNames() {
418    assert dir.hasReadLock();
419    if (!hasCreatedEncryptionZone()) {
420      return new String[0];
421    }
422    String[] ret = new String[encryptionZones.size()];
423    int index = 0;
424    for (Map.Entry<Long, EncryptionZoneInt> entry : encryptionZones
425        .entrySet()) {
426      ret[index++] = entry.getValue().getKeyName();
427    }
428    return ret;
429  }
430}