View Javadoc

1   /**
2    *
3    * Licensed to the Apache Software Foundation (ASF) under one
4    * or more contributor license agreements.  See the NOTICE file
5    * distributed with this work for additional information
6    * regarding copyright ownership.  The ASF licenses this file
7    * to you under the Apache License, Version 2.0 (the
8    * "License"); you may not use this file except in compliance
9    * with the License.  You may obtain a copy of the License at
10   *
11   *     http://www.apache.org/licenses/LICENSE-2.0
12   *
13   * Unless required by applicable law or agreed to in writing, software
14   * distributed under the License is distributed on an "AS IS" BASIS,
15   * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
16   * See the License for the specific language governing permissions and
17   * limitations under the License.
18   */
19  package org.apache.hadoop.hbase.zookeeper;
20  
21  import java.io.Closeable;
22  import java.io.IOException;
23  import java.util.ArrayList;
24  import java.util.List;
25  import java.util.concurrent.CopyOnWriteArrayList;
26  import java.util.concurrent.CountDownLatch;
27  
28  import org.apache.commons.logging.Log;
29  import org.apache.commons.logging.LogFactory;
30  import org.apache.hadoop.classification.InterfaceAudience;
31  import org.apache.hadoop.conf.Configuration;
32  import org.apache.hadoop.hbase.Abortable;
33  import org.apache.hadoop.hbase.HConstants;
34  import org.apache.hadoop.hbase.ZooKeeperConnectionException;
35  import org.apache.hadoop.hbase.util.Threads;
36  import org.apache.zookeeper.KeeperException;
37  import org.apache.zookeeper.WatchedEvent;
38  import org.apache.zookeeper.Watcher;
39  import org.apache.zookeeper.ZooDefs;
40  import org.apache.zookeeper.data.ACL;
41  
42  /**
43   * Acts as the single ZooKeeper Watcher.  One instance of this is instantiated
44   * for each Master, RegionServer, and client process.
45   *
46   * <p>This is the only class that implements {@link Watcher}.  Other internal
47   * classes which need to be notified of ZooKeeper events must register with
48   * the local instance of this watcher via {@link #registerListener}.
49   *
50   * <p>This class also holds and manages the connection to ZooKeeper.  Code to
51   * deal with connection related events and exceptions are handled here.
52   */
53  @InterfaceAudience.Private
54  public class ZooKeeperWatcher implements Watcher, Abortable, Closeable {
55    private static final Log LOG = LogFactory.getLog(ZooKeeperWatcher.class);
56  
57    // Identifier for this watcher (for logging only).  It is made of the prefix
58    // passed on construction and the zookeeper sessionid.
59    private String identifier;
60  
61    // zookeeper quorum
62    private String quorum;
63  
64    // zookeeper connection
65    private RecoverableZooKeeper recoverableZooKeeper;
66  
67    // abortable in case of zk failure
68    protected Abortable abortable;
69    // Used if abortable is null
70    private boolean aborted = false;
71  
72    // listeners to be notified
73    private final List<ZooKeeperListener> listeners =
74      new CopyOnWriteArrayList<ZooKeeperListener>();
75  
76    // Used by ZKUtil:waitForZKConnectionIfAuthenticating to wait for SASL
77    // negotiation to complete
78    public CountDownLatch saslLatch = new CountDownLatch(1);
79  
80    // node names
81  
82    // base znode for this cluster
83    public String baseZNode;
84    // znode containing location of server hosting meta region
85    public String metaServerZNode;
86    // znode containing ephemeral nodes of the regionservers
87    public String rsZNode;
88    // znode containing ephemeral nodes of the draining regionservers
89    public String drainingZNode;
90    // znode of currently active master
91    private String masterAddressZNode;
92    // znode of this master in backup master directory, if not the active master
93    public String backupMasterAddressesZNode;
94    // znode containing the current cluster state
95    public String clusterStateZNode;
96    // znode used for region transitioning and assignment
97    public String assignmentZNode;
98    // znode used for table disabling/enabling
99    public String tableZNode;
100   // znode containing the unique cluster ID
101   public String clusterIdZNode;
102   // znode used for log splitting work assignment
103   public String splitLogZNode;
104   // znode containing the state of the load balancer
105   public String balancerZNode;
106   // znode containing the lock for the tables
107   public String tableLockZNode;
108   // znode containing the state of recovering regions
109   public String recoveringRegionsZNode;
110   // znode containing namespace descriptors
111   public static String namespaceZNode = "namespace";
112 
113 
114   // Certain ZooKeeper nodes need to be world-readable
115   public static final ArrayList<ACL> CREATOR_ALL_AND_WORLD_READABLE =
116     new ArrayList<ACL>() { {
117       add(new ACL(ZooDefs.Perms.READ,ZooDefs.Ids.ANYONE_ID_UNSAFE));
118       add(new ACL(ZooDefs.Perms.ALL,ZooDefs.Ids.AUTH_IDS));
119     }};
120 
121   private final Configuration conf;
122 
123   private final Exception constructorCaller;
124 
125   /**
126    * Instantiate a ZooKeeper connection and watcher.
127    * @param identifier string that is passed to RecoverableZookeeper to be used as
128    * identifier for this instance. Use null for default.
129    * @throws IOException
130    * @throws ZooKeeperConnectionException
131    */
132   public ZooKeeperWatcher(Configuration conf, String identifier,
133       Abortable abortable) throws ZooKeeperConnectionException, IOException {
134     this(conf, identifier, abortable, false);
135   }
136 
137   /**
138    * Instantiate a ZooKeeper connection and watcher.
139    * @param conf
140    * @param identifier string that is passed to RecoverableZookeeper to be used as identifier for
141    *          this instance. Use null for default.
142    * @param abortable Can be null if there is on error there is no host to abort: e.g. client
143    *          context.
144    * @param canCreateBaseZNode
145    * @throws IOException
146    * @throws ZooKeeperConnectionException
147    */
148   public ZooKeeperWatcher(Configuration conf, String identifier,
149       Abortable abortable, boolean canCreateBaseZNode)
150   throws IOException, ZooKeeperConnectionException {
151     this.conf = conf;
152     // Capture a stack trace now.  Will print it out later if problem so we can
153     // distingush amongst the myriad ZKWs.
154     try {
155       throw new Exception("ZKW CONSTRUCTOR STACK TRACE FOR DEBUGGING");
156     } catch (Exception e) {
157       this.constructorCaller = e;
158     }
159     this.quorum = ZKConfig.getZKQuorumServersString(conf);
160     // Identifier will get the sessionid appended later below down when we
161     // handle the syncconnect event.
162     this.identifier = identifier;
163     this.abortable = abortable;
164     setNodeNames(conf);
165     this.recoverableZooKeeper = ZKUtil.connect(conf, quorum, this, identifier);
166     if (canCreateBaseZNode) {
167       createBaseZNodes();
168     }
169   }
170 
171   private void createBaseZNodes() throws ZooKeeperConnectionException {
172     try {
173       // Create all the necessary "directories" of znodes
174       ZKUtil.createWithParents(this, baseZNode);
175       if (conf.getBoolean("hbase.assignment.usezk", true)) {
176         ZKUtil.createAndFailSilent(this, assignmentZNode);
177       }
178       ZKUtil.createAndFailSilent(this, rsZNode);
179       ZKUtil.createAndFailSilent(this, drainingZNode);
180       ZKUtil.createAndFailSilent(this, tableZNode);
181       ZKUtil.createAndFailSilent(this, splitLogZNode);
182       ZKUtil.createAndFailSilent(this, backupMasterAddressesZNode);
183       ZKUtil.createAndFailSilent(this, tableLockZNode);
184       ZKUtil.createAndFailSilent(this, recoveringRegionsZNode);
185     } catch (KeeperException e) {
186       throw new ZooKeeperConnectionException(
187           prefix("Unexpected KeeperException creating base node"), e);
188     }
189   }
190 
191   @Override
192   public String toString() {
193     return this.identifier + ", quorum=" + quorum + ", baseZNode=" + baseZNode;
194   }
195 
196   /**
197    * Adds this instance's identifier as a prefix to the passed <code>str</code>
198    * @param str String to amend.
199    * @return A new string with this instance's identifier as prefix: e.g.
200    * if passed 'hello world', the returned string could be
201    */
202   public String prefix(final String str) {
203     return this.toString() + " " + str;
204   }
205 
206   /**
207    * Set the local variable node names using the specified configuration.
208    */
209   private void setNodeNames(Configuration conf) {
210     baseZNode = conf.get(HConstants.ZOOKEEPER_ZNODE_PARENT,
211         HConstants.DEFAULT_ZOOKEEPER_ZNODE_PARENT);
212     metaServerZNode = ZKUtil.joinZNode(baseZNode,
213         conf.get("zookeeper.znode.metaserver", "meta-region-server"));
214     rsZNode = ZKUtil.joinZNode(baseZNode,
215         conf.get("zookeeper.znode.rs", "rs"));
216     drainingZNode = ZKUtil.joinZNode(baseZNode,
217         conf.get("zookeeper.znode.draining.rs", "draining"));
218     masterAddressZNode = ZKUtil.joinZNode(baseZNode,
219         conf.get("zookeeper.znode.master", "master"));
220     backupMasterAddressesZNode = ZKUtil.joinZNode(baseZNode,
221         conf.get("zookeeper.znode.backup.masters", "backup-masters"));
222     clusterStateZNode = ZKUtil.joinZNode(baseZNode,
223         conf.get("zookeeper.znode.state", "running"));
224     assignmentZNode = ZKUtil.joinZNode(baseZNode,
225         conf.get("zookeeper.znode.unassigned", "region-in-transition"));
226     tableZNode = ZKUtil.joinZNode(baseZNode,
227         conf.get("zookeeper.znode.tableEnableDisable", "table"));
228     clusterIdZNode = ZKUtil.joinZNode(baseZNode,
229         conf.get("zookeeper.znode.clusterId", "hbaseid"));
230     splitLogZNode = ZKUtil.joinZNode(baseZNode,
231         conf.get("zookeeper.znode.splitlog", HConstants.SPLIT_LOGDIR_NAME));
232     balancerZNode = ZKUtil.joinZNode(baseZNode,
233         conf.get("zookeeper.znode.balancer", "balancer"));
234     tableLockZNode = ZKUtil.joinZNode(baseZNode,
235         conf.get("zookeeper.znode.tableLock", "table-lock"));
236     recoveringRegionsZNode = ZKUtil.joinZNode(baseZNode,
237         conf.get("zookeeper.znode.recovering.regions", "recovering-regions"));
238     namespaceZNode = ZKUtil.joinZNode(baseZNode,
239         conf.get("zookeeper.znode.namespace", "namespace"));
240   }
241 
242   /**
243    * Register the specified listener to receive ZooKeeper events.
244    * @param listener
245    */
246   public void registerListener(ZooKeeperListener listener) {
247     listeners.add(listener);
248   }
249 
250   /**
251    * Register the specified listener to receive ZooKeeper events and add it as
252    * the first in the list of current listeners.
253    * @param listener
254    */
255   public void registerListenerFirst(ZooKeeperListener listener) {
256     listeners.add(0, listener);
257   }
258 
259   public void unregisterListener(ZooKeeperListener listener) {
260     listeners.remove(listener);
261   }
262 
263   /**
264    * Clean all existing listeners
265    */
266   public void unregisterAllListeners() {
267     listeners.clear();
268   }
269 
270   /**
271    * Get a copy of current registered listeners
272    */
273   public List<ZooKeeperListener> getListeners() {
274     return new ArrayList<ZooKeeperListener>(listeners);
275   }
276 
277   /**
278    * @return The number of currently registered listeners
279    */
280   public int getNumberOfListeners() {
281     return listeners.size();
282   }
283 
284   /**
285    * Get the connection to ZooKeeper.
286    * @return connection reference to zookeeper
287    */
288   public RecoverableZooKeeper getRecoverableZooKeeper() {
289     return recoverableZooKeeper;
290   }
291 
292   public void reconnectAfterExpiration() throws IOException, KeeperException, InterruptedException {
293     recoverableZooKeeper.reconnectAfterExpiration();
294   }
295 
296   /**
297    * Get the quorum address of this instance.
298    * @return quorum string of this zookeeper connection instance
299    */
300   public String getQuorum() {
301     return quorum;
302   }
303 
304   /**
305    * Method called from ZooKeeper for events and connection status.
306    * <p>
307    * Valid events are passed along to listeners.  Connection status changes
308    * are dealt with locally.
309    */
310   @Override
311   public void process(WatchedEvent event) {
312     LOG.debug(prefix("Received ZooKeeper Event, " +
313         "type=" + event.getType() + ", " +
314         "state=" + event.getState() + ", " +
315         "path=" + event.getPath()));
316 
317     switch(event.getType()) {
318 
319       // If event type is NONE, this is a connection status change
320       case None: {
321         connectionEvent(event);
322         break;
323       }
324 
325       // Otherwise pass along to the listeners
326 
327       case NodeCreated: {
328         for(ZooKeeperListener listener : listeners) {
329           listener.nodeCreated(event.getPath());
330         }
331         break;
332       }
333 
334       case NodeDeleted: {
335         for(ZooKeeperListener listener : listeners) {
336           listener.nodeDeleted(event.getPath());
337         }
338         break;
339       }
340 
341       case NodeDataChanged: {
342         for(ZooKeeperListener listener : listeners) {
343           listener.nodeDataChanged(event.getPath());
344         }
345         break;
346       }
347 
348       case NodeChildrenChanged: {
349         for(ZooKeeperListener listener : listeners) {
350           listener.nodeChildrenChanged(event.getPath());
351         }
352         break;
353       }
354     }
355   }
356 
357   // Connection management
358 
359   /**
360    * Called when there is a connection-related event via the Watcher callback.
361    * <p>
362    * If Disconnected or Expired, this should shutdown the cluster. But, since
363    * we send a KeeperException.SessionExpiredException along with the abort
364    * call, it's possible for the Abortable to catch it and try to create a new
365    * session with ZooKeeper. This is what the client does in HCM.
366    * <p>
367    * @param event
368    */
369   private void connectionEvent(WatchedEvent event) {
370     switch(event.getState()) {
371       case SyncConnected:
372         // Now, this callback can be invoked before the this.zookeeper is set.
373         // Wait a little while.
374         long finished = System.currentTimeMillis() +
375           this.conf.getLong("hbase.zookeeper.watcher.sync.connected.wait", 2000);
376         while (System.currentTimeMillis() < finished) {
377           Threads.sleep(1);
378           if (this.recoverableZooKeeper != null) break;
379         }
380         if (this.recoverableZooKeeper == null) {
381           LOG.error("ZK is null on connection event -- see stack trace " +
382             "for the stack trace when constructor was called on this zkw",
383             this.constructorCaller);
384           throw new NullPointerException("ZK is null");
385         }
386         this.identifier = this.identifier + "-0x" +
387           Long.toHexString(this.recoverableZooKeeper.getSessionId());
388         // Update our identifier.  Otherwise ignore.
389         LOG.debug(this.identifier + " connected");
390         break;
391 
392       // Abort the server if Disconnected or Expired
393       case Disconnected:
394         LOG.debug(prefix("Received Disconnected from ZooKeeper, ignoring"));
395         break;
396 
397       case Expired:
398         String msg = prefix(this.identifier + " received expired from " +
399           "ZooKeeper, aborting");
400         // TODO: One thought is to add call to ZooKeeperListener so say,
401         // ZooKeeperNodeTracker can zero out its data values.
402         if (this.abortable != null) {
403           this.abortable.abort(msg, new KeeperException.SessionExpiredException());
404         }
405         break;
406 
407       case ConnectedReadOnly:
408       case SaslAuthenticated:
409       case AuthFailed:
410         break;
411 
412       default:
413         throw new IllegalStateException("Received event is not valid: " + event.getState());
414     }
415   }
416 
417   /**
418    * Forces a synchronization of this ZooKeeper client connection.
419    * <p>
420    * Executing this method before running other methods will ensure that the
421    * subsequent operations are up-to-date and consistent as of the time that
422    * the sync is complete.
423    * <p>
424    * This is used for compareAndSwap type operations where we need to read the
425    * data of an existing node and delete or transition that node, utilizing the
426    * previously read version and data.  We want to ensure that the version read
427    * is up-to-date from when we begin the operation.
428    */
429   public void sync(String path) throws KeeperException {
430     this.recoverableZooKeeper.sync(path, null, null);
431   }
432 
433   /**
434    * Handles KeeperExceptions in client calls.
435    * <p>
436    * This may be temporary but for now this gives one place to deal with these.
437    * <p>
438    * TODO: Currently this method rethrows the exception to let the caller handle
439    * <p>
440    * @param ke
441    * @throws KeeperException
442    */
443   public void keeperException(KeeperException ke)
444   throws KeeperException {
445     LOG.error(prefix("Received unexpected KeeperException, re-throwing exception"), ke);
446     throw ke;
447   }
448 
449   /**
450    * Handles InterruptedExceptions in client calls.
451    * <p>
452    * This may be temporary but for now this gives one place to deal with these.
453    * <p>
454    * TODO: Currently, this method does nothing.
455    *       Is this ever expected to happen?  Do we abort or can we let it run?
456    *       Maybe this should be logged as WARN?  It shouldn't happen?
457    * <p>
458    * @param ie
459    */
460   public void interruptedException(InterruptedException ie) {
461     LOG.debug(prefix("Received InterruptedException, doing nothing here"), ie);
462     // At least preserver interrupt.
463     Thread.currentThread().interrupt();
464     // no-op
465   }
466 
467   /**
468    * Close the connection to ZooKeeper.
469    *
470    * @throws InterruptedException
471    */
472   public void close() {
473     try {
474       if (recoverableZooKeeper != null) {
475         recoverableZooKeeper.close();
476       }
477     } catch (InterruptedException e) {
478       Thread.currentThread().interrupt();
479     }
480   }
481 
482   public Configuration getConfiguration() {
483     return conf;
484   }
485 
486   @Override
487   public void abort(String why, Throwable e) {
488     if (this.abortable != null) this.abortable.abort(why, e);
489     else this.aborted = true;
490   }
491 
492   @Override
493   public boolean isAborted() {
494     return this.abortable == null? this.aborted: this.abortable.isAborted();
495   }
496 
497   /**
498    * @return Path to the currently active master.
499    */
500   public String getMasterAddressZNode() {
501     return this.masterAddressZNode;
502   }
503 
504 }