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