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       if (conf.getBoolean("hbase.assignment.usezk", true)) {
175         ZKUtil.createAndFailSilent(this, assignmentZNode);
176       }
177       ZKUtil.createAndFailSilent(this, rsZNode);
178       ZKUtil.createAndFailSilent(this, drainingZNode);
179       ZKUtil.createAndFailSilent(this, tableZNode);
180       ZKUtil.createAndFailSilent(this, splitLogZNode);
181       ZKUtil.createAndFailSilent(this, backupMasterAddressesZNode);
182       ZKUtil.createAndFailSilent(this, tableLockZNode);
183       ZKUtil.createAndFailSilent(this, recoveringRegionsZNode);
184     } catch (KeeperException e) {
185       throw new ZooKeeperConnectionException(
186           prefix("Unexpected KeeperException creating base node"), e);
187     }
188   }
189 
190   @Override
191   public String toString() {
192     return this.identifier + ", quorum=" + quorum + ", baseZNode=" + baseZNode;
193   }
194 
195   /**
196    * Adds this instance's identifier as a prefix to the passed <code>str</code>
197    * @param str String to amend.
198    * @return A new string with this instance's identifier as prefix: e.g.
199    * if passed 'hello world', the returned string could be
200    */
201   public String prefix(final String str) {
202     return this.toString() + " " + str;
203   }
204 
205   /**
206    * Set the local variable node names using the specified configuration.
207    */
208   private void setNodeNames(Configuration conf) {
209     baseZNode = conf.get(HConstants.ZOOKEEPER_ZNODE_PARENT,
210         HConstants.DEFAULT_ZOOKEEPER_ZNODE_PARENT);
211     metaServerZNode = ZKUtil.joinZNode(baseZNode,
212         conf.get("zookeeper.znode.metaserver", "meta-region-server"));
213     rsZNode = ZKUtil.joinZNode(baseZNode,
214         conf.get("zookeeper.znode.rs", "rs"));
215     drainingZNode = ZKUtil.joinZNode(baseZNode,
216         conf.get("zookeeper.znode.draining.rs", "draining"));
217     masterAddressZNode = ZKUtil.joinZNode(baseZNode,
218         conf.get("zookeeper.znode.master", "master"));
219     backupMasterAddressesZNode = ZKUtil.joinZNode(baseZNode,
220         conf.get("zookeeper.znode.backup.masters", "backup-masters"));
221     clusterStateZNode = ZKUtil.joinZNode(baseZNode,
222         conf.get("zookeeper.znode.state", "running"));
223     assignmentZNode = ZKUtil.joinZNode(baseZNode,
224         conf.get("zookeeper.znode.unassigned", "region-in-transition"));
225     tableZNode = ZKUtil.joinZNode(baseZNode,
226         conf.get("zookeeper.znode.tableEnableDisable", "table"));
227     clusterIdZNode = ZKUtil.joinZNode(baseZNode,
228         conf.get("zookeeper.znode.clusterId", "hbaseid"));
229     splitLogZNode = ZKUtil.joinZNode(baseZNode,
230         conf.get("zookeeper.znode.splitlog", HConstants.SPLIT_LOGDIR_NAME));
231     balancerZNode = ZKUtil.joinZNode(baseZNode,
232         conf.get("zookeeper.znode.balancer", "balancer"));
233     tableLockZNode = ZKUtil.joinZNode(baseZNode,
234         conf.get("zookeeper.znode.tableLock", "table-lock"));
235     recoveringRegionsZNode = ZKUtil.joinZNode(baseZNode,
236         conf.get("zookeeper.znode.recovering.regions", "recovering-regions"));
237     namespaceZNode = ZKUtil.joinZNode(baseZNode,
238         conf.get("zookeeper.znode.namespace", "namespace"));
239   }
240 
241   /**
242    * Register the specified listener to receive ZooKeeper events.
243    * @param listener
244    */
245   public void registerListener(ZooKeeperListener listener) {
246     listeners.add(listener);
247   }
248 
249   /**
250    * Register the specified listener to receive ZooKeeper events and add it as
251    * the first in the list of current listeners.
252    * @param listener
253    */
254   public void registerListenerFirst(ZooKeeperListener listener) {
255     listeners.add(0, listener);
256   }
257 
258   public void unregisterListener(ZooKeeperListener listener) {
259     listeners.remove(listener);
260   }
261 
262   /**
263    * Clean all existing listeners
264    */
265   public void unregisterAllListeners() {
266     listeners.clear();
267   }
268 
269   /**
270    * Get a copy of current registered listeners
271    */
272   public List<ZooKeeperListener> getListeners() {
273     return new ArrayList<ZooKeeperListener>(listeners);
274   }
275 
276   /**
277    * @return The number of currently registered listeners
278    */
279   public int getNumberOfListeners() {
280     return listeners.size();
281   }
282 
283   /**
284    * Get the connection to ZooKeeper.
285    * @return connection reference to zookeeper
286    */
287   public RecoverableZooKeeper getRecoverableZooKeeper() {
288     return recoverableZooKeeper;
289   }
290 
291   public void reconnectAfterExpiration() throws IOException, InterruptedException {
292     recoverableZooKeeper.reconnectAfterExpiration();
293   }
294 
295   /**
296    * Get the quorum address of this instance.
297    * @return quorum string of this zookeeper connection instance
298    */
299   public String getQuorum() {
300     return quorum;
301   }
302 
303   /**
304    * Method called from ZooKeeper for events and connection status.
305    * <p>
306    * Valid events are passed along to listeners.  Connection status changes
307    * are dealt with locally.
308    */
309   @Override
310   public void process(WatchedEvent event) {
311     LOG.debug(prefix("Received ZooKeeper Event, " +
312         "type=" + event.getType() + ", " +
313         "state=" + event.getState() + ", " +
314         "path=" + event.getPath()));
315 
316     switch(event.getType()) {
317 
318       // If event type is NONE, this is a connection status change
319       case None: {
320         connectionEvent(event);
321         break;
322       }
323 
324       // Otherwise pass along to the listeners
325 
326       case NodeCreated: {
327         for(ZooKeeperListener listener : listeners) {
328           listener.nodeCreated(event.getPath());
329         }
330         break;
331       }
332 
333       case NodeDeleted: {
334         for(ZooKeeperListener listener : listeners) {
335           listener.nodeDeleted(event.getPath());
336         }
337         break;
338       }
339 
340       case NodeDataChanged: {
341         for(ZooKeeperListener listener : listeners) {
342           listener.nodeDataChanged(event.getPath());
343         }
344         break;
345       }
346 
347       case NodeChildrenChanged: {
348         for(ZooKeeperListener listener : listeners) {
349           listener.nodeChildrenChanged(event.getPath());
350         }
351         break;
352       }
353     }
354   }
355 
356   // Connection management
357 
358   /**
359    * Called when there is a connection-related event via the Watcher callback.
360    * <p>
361    * If Disconnected or Expired, this should shutdown the cluster. But, since
362    * we send a KeeperException.SessionExpiredException along with the abort
363    * call, it's possible for the Abortable to catch it and try to create a new
364    * session with ZooKeeper. This is what the client does in HCM.
365    * <p>
366    * @param event
367    */
368   private void connectionEvent(WatchedEvent event) {
369     switch(event.getState()) {
370       case SyncConnected:
371         // Now, this callback can be invoked before the this.zookeeper is set.
372         // Wait a little while.
373         long finished = System.currentTimeMillis() +
374           this.conf.getLong("hbase.zookeeper.watcher.sync.connected.wait", 2000);
375         while (System.currentTimeMillis() < finished) {
376           try {
377             Thread.sleep(1);
378           } catch (InterruptedException e) {
379             LOG.warn("Interrupted while sleeping");
380             throw new RuntimeException("Interrupted while waiting for" +
381                 " recoverableZooKeeper is set");
382           }
383           if (this.recoverableZooKeeper != null) break;
384         }
385 
386         if (this.recoverableZooKeeper == null) {
387           LOG.error("ZK is null on connection event -- see stack trace " +
388             "for the stack trace when constructor was called on this zkw",
389             this.constructorCaller);
390           throw new NullPointerException("ZK is null");
391         }
392         this.identifier = this.identifier + "-0x" +
393           Long.toHexString(this.recoverableZooKeeper.getSessionId());
394         // Update our identifier.  Otherwise ignore.
395         LOG.debug(this.identifier + " connected");
396         break;
397 
398       // Abort the server if Disconnected or Expired
399       case Disconnected:
400         LOG.debug(prefix("Received Disconnected from ZooKeeper, ignoring"));
401         break;
402 
403       case Expired:
404         String msg = prefix(this.identifier + " received expired from " +
405           "ZooKeeper, aborting");
406         // TODO: One thought is to add call to ZooKeeperListener so say,
407         // ZooKeeperNodeTracker can zero out its data values.
408         if (this.abortable != null) {
409           this.abortable.abort(msg, new KeeperException.SessionExpiredException());
410         }
411         break;
412 
413       case ConnectedReadOnly:
414       case SaslAuthenticated:
415       case AuthFailed:
416         break;
417 
418       default:
419         throw new IllegalStateException("Received event is not valid: " + event.getState());
420     }
421   }
422 
423   /**
424    * Forces a synchronization of this ZooKeeper client connection.
425    * <p>
426    * Executing this method before running other methods will ensure that the
427    * subsequent operations are up-to-date and consistent as of the time that
428    * the sync is complete.
429    * <p>
430    * This is used for compareAndSwap type operations where we need to read the
431    * data of an existing node and delete or transition that node, utilizing the
432    * previously read version and data.  We want to ensure that the version read
433    * is up-to-date from when we begin the operation.
434    */
435   public void sync(String path) {
436     this.recoverableZooKeeper.sync(path, null, null);
437   }
438 
439   /**
440    * Handles KeeperExceptions in client calls.
441    * <p>
442    * This may be temporary but for now this gives one place to deal with these.
443    * <p>
444    * TODO: Currently this method rethrows the exception to let the caller handle
445    * <p>
446    * @param ke
447    * @throws KeeperException
448    */
449   public void keeperException(KeeperException ke)
450   throws KeeperException {
451     LOG.error(prefix("Received unexpected KeeperException, re-throwing exception"), ke);
452     throw ke;
453   }
454 
455   /**
456    * Handles InterruptedExceptions in client calls.
457    * <p>
458    * This may be temporary but for now this gives one place to deal with these.
459    * <p>
460    * TODO: Currently, this method does nothing.
461    *       Is this ever expected to happen?  Do we abort or can we let it run?
462    *       Maybe this should be logged as WARN?  It shouldn't happen?
463    * <p>
464    * @param ie
465    */
466   public void interruptedException(InterruptedException ie) {
467     LOG.debug(prefix("Received InterruptedException, doing nothing here"), ie);
468     // At least preserver interrupt.
469     Thread.currentThread().interrupt();
470     // no-op
471   }
472 
473   /**
474    * Close the connection to ZooKeeper.
475    *
476    * @throws InterruptedException
477    */
478   public void close() {
479     try {
480       if (recoverableZooKeeper != null) {
481         recoverableZooKeeper.close();
482       }
483     } catch (InterruptedException e) {
484       Thread.currentThread().interrupt();
485     }
486   }
487 
488   public Configuration getConfiguration() {
489     return conf;
490   }
491 
492   @Override
493   public void abort(String why, Throwable e) {
494     if (this.abortable != null) this.abortable.abort(why, e);
495     else this.aborted = true;
496   }
497 
498   @Override
499   public boolean isAborted() {
500     return this.abortable == null? this.aborted: this.abortable.isAborted();
501   }
502 
503   /**
504    * @return Path to the currently active master.
505    */
506   public String getMasterAddressZNode() {
507     return this.masterAddressZNode;
508   }
509 
510 }