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