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