View Javadoc

1   /*
2    * Licensed to the Apache Software Foundation (ASF) under one
3    * or more contributor license agreements.  See the NOTICE file
4    * distributed with this work for additional information
5    * regarding copyright ownership.  The ASF licenses this file
6    * to you under the Apache License, Version 2.0 (the
7    * "License"); you may not use this file except in compliance
8    * with the License.  You may obtain a copy of the License at
9    *
10   *     http://www.apache.org/licenses/LICENSE-2.0
11   *
12   * Unless required by applicable law or agreed to in writing, software
13   * distributed under the License is distributed on an "AS IS" BASIS,
14   * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15   * See the License for the specific language governing permissions and
16   * limitations under the License.
17   */
18  
19  package org.apache.hadoop.hbase.security.access;
20  
21  import org.apache.commons.logging.Log;
22  import org.apache.commons.logging.LogFactory;
23  import org.apache.hadoop.hbase.classification.InterfaceAudience;
24  import org.apache.hadoop.conf.Configuration;
25  import org.apache.hadoop.hbase.TableName;
26  import org.apache.hadoop.hbase.util.Bytes;
27  import org.apache.hadoop.hbase.zookeeper.ZKUtil;
28  import org.apache.hadoop.hbase.zookeeper.ZooKeeperListener;
29  import org.apache.hadoop.hbase.zookeeper.ZooKeeperWatcher;
30  import org.apache.zookeeper.KeeperException;
31  
32  import java.io.IOException;
33  import java.util.List;
34  import java.util.concurrent.CountDownLatch;
35  
36  /**
37   * Handles synchronization of access control list entries and updates
38   * throughout all nodes in the cluster.  The {@link AccessController} instance
39   * on the {@code _acl_} table regions, creates a znode for each table as
40   * {@code /hbase/acl/tablename}, with the znode data containing a serialized
41   * list of the permissions granted for the table.  The {@code AccessController}
42   * instances on all other cluster hosts watch the znodes for updates, which
43   * trigger updates in the {@link TableAuthManager} permission cache.
44   */
45  @InterfaceAudience.Private
46  public class ZKPermissionWatcher extends ZooKeeperListener {
47    private static final Log LOG = LogFactory.getLog(ZKPermissionWatcher.class);
48    // parent node for permissions lists
49    static final String ACL_NODE = "acl";
50    TableAuthManager authManager;
51    String aclZNode;
52    CountDownLatch initialized = new CountDownLatch(1);
53  
54    public ZKPermissionWatcher(ZooKeeperWatcher watcher,
55        TableAuthManager authManager, Configuration conf) {
56      super(watcher);
57      this.authManager = authManager;
58      String aclZnodeParent = conf.get("zookeeper.znode.acl.parent", ACL_NODE);
59      this.aclZNode = ZKUtil.joinZNode(watcher.baseZNode, aclZnodeParent);
60    }
61  
62    public void start() throws KeeperException {
63      try {
64        watcher.registerListener(this);
65        if (ZKUtil.watchAndCheckExists(watcher, aclZNode)) {
66          List<ZKUtil.NodeAndData> existing =
67              ZKUtil.getChildDataAndWatchForNewChildren(watcher, aclZNode);
68          if (existing != null) {
69            refreshNodes(existing);
70          }
71        }
72      } finally {
73        initialized.countDown();
74      }
75    }
76  
77    private void waitUntilStarted() {
78      try {
79        initialized.await();
80      } catch (InterruptedException e) {
81        LOG.warn("Interrupted while waiting", e);
82        Thread.currentThread().interrupt();
83      }
84    }
85  
86    @Override
87    public void nodeCreated(String path) {
88      waitUntilStarted();
89      if (path.equals(aclZNode)) {
90        try {
91          List<ZKUtil.NodeAndData> nodes =
92              ZKUtil.getChildDataAndWatchForNewChildren(watcher, aclZNode);
93          refreshNodes(nodes);
94        } catch (KeeperException ke) {
95          LOG.error("Error reading data from zookeeper", ke);
96          // only option is to abort
97          watcher.abort("Zookeeper error obtaining acl node children", ke);
98        }
99      }
100   }
101 
102   @Override
103   public void nodeDeleted(String path) {
104     waitUntilStarted();
105     if (aclZNode.equals(ZKUtil.getParent(path))) {
106       String table = ZKUtil.getNodeName(path);
107       if(AccessControlLists.isNamespaceEntry(table)) {
108         authManager.removeNamespace(Bytes.toBytes(table));
109       } else {
110         authManager.removeTable(TableName.valueOf(table));
111       }
112     }
113   }
114 
115   @Override
116   public void nodeDataChanged(String path) {
117     waitUntilStarted();
118     if (aclZNode.equals(ZKUtil.getParent(path))) {
119       // update cache on an existing table node
120       String entry = ZKUtil.getNodeName(path);
121       try {
122         byte[] data = ZKUtil.getDataAndWatch(watcher, path);
123         refreshAuthManager(entry, data);
124       } catch (KeeperException ke) {
125         LOG.error("Error reading data from zookeeper for node " + entry, ke);
126         // only option is to abort
127         watcher.abort("Zookeeper error getting data for node " + entry, ke);
128       } catch (IOException ioe) {
129         LOG.error("Error reading permissions writables", ioe);
130       }
131     }
132   }
133 
134   @Override
135   public void nodeChildrenChanged(String path) {
136     waitUntilStarted();
137     if (path.equals(aclZNode)) {
138       // table permissions changed
139       try {
140         List<ZKUtil.NodeAndData> nodes =
141             ZKUtil.getChildDataAndWatchForNewChildren(watcher, aclZNode);
142         refreshNodes(nodes);
143       } catch (KeeperException ke) {
144         LOG.error("Error reading data from zookeeper for path "+path, ke);
145         watcher.abort("Zookeeper error get node children for path "+path, ke);
146       }
147     }
148   }
149 
150   private void refreshNodes(List<ZKUtil.NodeAndData> nodes) {
151     for (ZKUtil.NodeAndData n : nodes) {
152       if (n.isEmpty()) continue;
153       String path = n.getNode();
154       String entry = (ZKUtil.getNodeName(path));
155       try {
156         refreshAuthManager(entry, n.getData());
157       } catch (IOException ioe) {
158         LOG.error("Failed parsing permissions for table '" + entry +
159             "' from zk", ioe);
160       }
161     }
162   }
163 
164   private void refreshAuthManager(String entry, byte[] nodeData) throws IOException {
165     if (LOG.isDebugEnabled()) {
166       LOG.debug("Updating permissions cache from node "+entry+" with data: "+
167           Bytes.toStringBinary(nodeData));
168     }
169     if(AccessControlLists.isNamespaceEntry(entry)) {
170       authManager.refreshNamespaceCacheFromWritable(
171           AccessControlLists.fromNamespaceEntry(entry), nodeData);
172     } else {
173       authManager.refreshTableCacheFromWritable(TableName.valueOf(entry), nodeData);
174     }
175   }
176 
177   /***
178    * Write a table's access controls to the permissions mirror in zookeeper
179    * @param entry
180    * @param permsData
181    */
182   public void writeToZookeeper(byte[] entry, byte[] permsData) {
183     String entryName = Bytes.toString(entry);
184     String zkNode = ZKUtil.joinZNode(watcher.baseZNode, ACL_NODE);
185     zkNode = ZKUtil.joinZNode(zkNode, entryName);
186 
187     try {
188       ZKUtil.createWithParents(watcher, zkNode);
189       ZKUtil.updateExistingNodeData(watcher, zkNode, permsData, -1);
190     } catch (KeeperException e) {
191       LOG.error("Failed updating permissions for entry '" +
192           entryName + "'", e);
193       watcher.abort("Failed writing node "+zkNode+" to zookeeper", e);
194     }
195   }
196 
197   /***
198    * Delete the acl notify node of table
199    * @param tableName
200    */
201   public void deleteTableACLNode(final TableName tableName) {
202     String zkNode = ZKUtil.joinZNode(watcher.baseZNode, ACL_NODE);
203     zkNode = ZKUtil.joinZNode(zkNode, tableName.getNameAsString());
204 
205     try {
206       ZKUtil.deleteNode(watcher, zkNode);
207     } catch (KeeperException.NoNodeException e) {
208       LOG.warn("No acl notify node of table '" + tableName + "'");
209     } catch (KeeperException e) {
210       LOG.error("Failed deleting acl node of table '" + tableName + "'", e);
211       watcher.abort("Failed deleting node " + zkNode, e);
212     }
213   }
214 
215   /***
216    * Delete the acl notify node of namespace
217    */
218   public void deleteNamespaceACLNode(final String namespace) {
219     String zkNode = ZKUtil.joinZNode(watcher.baseZNode, ACL_NODE);
220     zkNode = ZKUtil.joinZNode(zkNode, AccessControlLists.NAMESPACE_PREFIX + namespace);
221 
222     try {
223       ZKUtil.deleteNode(watcher, zkNode);
224     } catch (KeeperException.NoNodeException e) {
225       LOG.warn("No acl notify node of namespace '" + namespace + "'");
226     } catch (KeeperException e) {
227       LOG.error("Failed deleting acl node of namespace '" + namespace + "'", e);
228       watcher.abort("Failed deleting node " + zkNode, e);
229     }
230   }
231 }