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 org.apache.commons.logging.Log;
22  import org.apache.commons.logging.LogFactory;
23  import org.apache.hadoop.classification.InterfaceAudience;
24  import org.apache.hadoop.classification.InterfaceStability;
25  import org.apache.hadoop.conf.Configuration;
26  import org.apache.hadoop.hbase.Abortable;
27  import org.apache.hadoop.hbase.HConstants;
28  import org.apache.hadoop.hbase.exceptions.ZooKeeperConnectionException;
29  import org.apache.hadoop.hbase.util.Threads;
30  import org.apache.zookeeper.KeeperException;
31  import org.apache.zookeeper.WatchedEvent;
32  import org.apache.zookeeper.Watcher;
33  import org.apache.zookeeper.ZooDefs;
34  import org.apache.zookeeper.data.ACL;
35  
36  import java.io.Closeable;
37  import java.io.IOException;
38  import java.util.ArrayList;
39  import java.util.List;
40  import java.util.concurrent.CopyOnWriteArrayList;
41  import java.util.concurrent.CountDownLatch;
42  
43  /**
44   * Acts as the single ZooKeeper Watcher.  One instance of this is instantiated
45   * for each Master, RegionServer, and client process.
46   *
47   * <p>This is the only class that implements {@link Watcher}.  Other internal
48   * classes which need to be notified of ZooKeeper events must register with
49   * the local instance of this watcher via {@link #registerListener}.
50   *
51   * <p>This class also holds and manages the connection to ZooKeeper.  Code to
52   * deal with connection related events and exceptions are handled here.
53   */
54  @InterfaceAudience.Public
55  @InterfaceStability.Evolving
56  public class ZooKeeperWatcher implements Watcher, Abortable, Closeable {
57    private static final Log LOG = LogFactory.getLog(ZooKeeperWatcher.class);
58  
59    // Identifier for this watcher (for logging only).  It is made of the prefix
60    // passed on construction and the zookeeper sessionid.
61    private String identifier;
62  
63    // zookeeper quorum
64    private String quorum;
65  
66    // zookeeper connection
67    private RecoverableZooKeeper recoverableZooKeeper;
68  
69    // abortable in case of zk failure
70    protected Abortable abortable;
71    // Used if abortable is null
72    private boolean aborted = false;
73  
74    // listeners to be notified
75    private final List<ZooKeeperListener> listeners =
76      new CopyOnWriteArrayList<ZooKeeperListener>();
77  
78    // Used by ZKUtil:waitForZKConnectionIfAuthenticating to wait for SASL
79    // negotiation to complete
80    public CountDownLatch saslLatch = new CountDownLatch(1);
81  
82    // node names
83  
84    // base znode for this cluster
85    public String baseZNode;
86    // znode containing location of server hosting meta region
87    public String metaServerZNode;
88    // znode containing ephemeral nodes of the regionservers
89    public String rsZNode;
90    // znode containing ephemeral nodes of the draining regionservers
91    public String drainingZNode;
92    // znode of currently active master
93    private String masterAddressZNode;
94    // znode of this master in backup master directory, if not the active master
95    public String backupMasterAddressesZNode;
96    // znode containing the current cluster state
97    public String clusterStateZNode;
98    // znode used for region transitioning and assignment
99    public String assignmentZNode;
100   // znode used for table disabling/enabling
101   public String tableZNode;
102   // znode containing the unique cluster ID
103   public String clusterIdZNode;
104   // znode used for log splitting work assignment
105   public String splitLogZNode;
106   // znode containing the state of the load balancer
107   public String balancerZNode;
108   // znode containing the lock for the tables
109   public String tableLockZNode;
110   // znode containing the state of recovering regions
111   public String recoveringRegionsZNode;
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;
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   }
236 
237   /**
238    * Register the specified listener to receive ZooKeeper events.
239    * @param listener
240    */
241   public void registerListener(ZooKeeperListener listener) {
242     listeners.add(listener);
243   }
244 
245   /**
246    * Register the specified listener to receive ZooKeeper events and add it as
247    * the first in the list of current listeners.
248    * @param listener
249    */
250   public void registerListenerFirst(ZooKeeperListener listener) {
251     listeners.add(0, listener);
252   }
253 
254   public void unregisterListener(ZooKeeperListener listener) {
255     listeners.remove(listener);
256   }
257 
258   /**
259    * Clean all existing listeners
260    */
261   public void unregisterAllListeners() {
262     listeners.clear();
263   }
264 
265   /**
266    * @return The number of currently registered listeners
267    */
268   public int getNumberOfListeners() {
269     return listeners.size();
270   }
271 
272   /**
273    * Get the connection to ZooKeeper.
274    * @return connection reference to zookeeper
275    */
276   public RecoverableZooKeeper getRecoverableZooKeeper() {
277     return recoverableZooKeeper;
278   }
279 
280   public void reconnectAfterExpiration() throws IOException, InterruptedException {
281     recoverableZooKeeper.reconnectAfterExpiration();
282   }
283 
284   /**
285    * Get the quorum address of this instance.
286    * @return quorum string of this zookeeper connection instance
287    */
288   public String getQuorum() {
289     return quorum;
290   }
291 
292   /**
293    * Method called from ZooKeeper for events and connection status.
294    * <p>
295    * Valid events are passed along to listeners.  Connection status changes
296    * are dealt with locally.
297    */
298   @Override
299   public void process(WatchedEvent event) {
300     LOG.debug(prefix("Received ZooKeeper Event, " +
301         "type=" + event.getType() + ", " +
302         "state=" + event.getState() + ", " +
303         "path=" + event.getPath()));
304 
305     switch(event.getType()) {
306 
307       // If event type is NONE, this is a connection status change
308       case None: {
309         connectionEvent(event);
310         break;
311       }
312 
313       // Otherwise pass along to the listeners
314 
315       case NodeCreated: {
316         for(ZooKeeperListener listener : listeners) {
317           listener.nodeCreated(event.getPath());
318         }
319         break;
320       }
321 
322       case NodeDeleted: {
323         for(ZooKeeperListener listener : listeners) {
324           listener.nodeDeleted(event.getPath());
325         }
326         break;
327       }
328 
329       case NodeDataChanged: {
330         for(ZooKeeperListener listener : listeners) {
331           listener.nodeDataChanged(event.getPath());
332         }
333         break;
334       }
335 
336       case NodeChildrenChanged: {
337         for(ZooKeeperListener listener : listeners) {
338           listener.nodeChildrenChanged(event.getPath());
339         }
340         break;
341       }
342     }
343   }
344 
345   // Connection management
346 
347   /**
348    * Called when there is a connection-related event via the Watcher callback.
349    * <p>
350    * If Disconnected or Expired, this should shutdown the cluster. But, since
351    * we send a KeeperException.SessionExpiredException along with the abort
352    * call, it's possible for the Abortable to catch it and try to create a new
353    * session with ZooKeeper. This is what the client does in HCM.
354    * <p>
355    * @param event
356    */
357   private void connectionEvent(WatchedEvent event) {
358     switch(event.getState()) {
359       case SyncConnected:
360         // Now, this callback can be invoked before the this.zookeeper is set.
361         // Wait a little while.
362         long finished = System.currentTimeMillis() +
363           this.conf.getLong("hbase.zookeeper.watcher.sync.connected.wait", 2000);
364         while (System.currentTimeMillis() < finished) {
365           Threads.sleep(1);
366           if (this.recoverableZooKeeper != null) break;
367         }
368         if (this.recoverableZooKeeper == null) {
369           LOG.error("ZK is null on connection event -- see stack trace " +
370             "for the stack trace when constructor was called on this zkw",
371             this.constructorCaller);
372           throw new NullPointerException("ZK is null");
373         }
374         this.identifier = this.identifier + "-0x" +
375           Long.toHexString(this.recoverableZooKeeper.getSessionId());
376         // Update our identifier.  Otherwise ignore.
377         LOG.debug(this.identifier + " connected");
378         break;
379 
380       // Abort the server if Disconnected or Expired
381       case Disconnected:
382         LOG.debug(prefix("Received Disconnected from ZooKeeper, ignoring"));
383         break;
384 
385       case Expired:
386         String msg = prefix(this.identifier + " received expired from " +
387           "ZooKeeper, aborting");
388         // TODO: One thought is to add call to ZooKeeperListener so say,
389         // ZooKeeperNodeTracker can zero out its data values.
390         if (this.abortable != null) {
391           this.abortable.abort(msg, new KeeperException.SessionExpiredException());
392         }
393         break;
394 
395       case ConnectedReadOnly:
396         break;
397 
398       default:
399         throw new IllegalStateException("Received event is not valid.");
400     }
401   }
402 
403   /**
404    * Forces a synchronization of this ZooKeeper client connection.
405    * <p>
406    * Executing this method before running other methods will ensure that the
407    * subsequent operations are up-to-date and consistent as of the time that
408    * the sync is complete.
409    * <p>
410    * This is used for compareAndSwap type operations where we need to read the
411    * data of an existing node and delete or transition that node, utilizing the
412    * previously read version and data.  We want to ensure that the version read
413    * is up-to-date from when we begin the operation.
414    */
415   public void sync(String path) {
416     this.recoverableZooKeeper.sync(path, null, null);
417   }
418 
419   /**
420    * Handles KeeperExceptions in client calls.
421    * <p>
422    * This may be temporary but for now this gives one place to deal with these.
423    * <p>
424    * TODO: Currently this method rethrows the exception to let the caller handle
425    * <p>
426    * @param ke
427    * @throws KeeperException
428    */
429   public void keeperException(KeeperException ke)
430   throws KeeperException {
431     LOG.error(prefix("Received unexpected KeeperException, re-throwing exception"), ke);
432     throw ke;
433   }
434 
435   /**
436    * Handles InterruptedExceptions in client calls.
437    * <p>
438    * This may be temporary but for now this gives one place to deal with these.
439    * <p>
440    * TODO: Currently, this method does nothing.
441    *       Is this ever expected to happen?  Do we abort or can we let it run?
442    *       Maybe this should be logged as WARN?  It shouldn't happen?
443    * <p>
444    * @param ie
445    */
446   public void interruptedException(InterruptedException ie) {
447     LOG.debug(prefix("Received InterruptedException, doing nothing here"), ie);
448     // At least preserver interrupt.
449     Thread.currentThread().interrupt();
450     // no-op
451   }
452 
453   /**
454    * Close the connection to ZooKeeper.
455    *
456    * @throws InterruptedException
457    */
458   public void close() {
459     try {
460       if (recoverableZooKeeper != null) {
461         recoverableZooKeeper.close();
462       }
463     } catch (InterruptedException e) {
464       Thread.currentThread().interrupt();
465     }
466   }
467 
468   public Configuration getConfiguration() {
469     return conf;
470   }
471 
472   @Override
473   public void abort(String why, Throwable e) {
474     if (this.abortable != null) this.abortable.abort(why, e);
475     else this.aborted = true;
476   }
477 
478   @Override
479   public boolean isAborted() {
480     return this.abortable == null? this.aborted: this.abortable.isAborted();
481   }
482 
483   /**
484    * @return Path to the currently active master.
485    */
486   public String getMasterAddressZNode() {
487     return this.masterAddressZNode;
488   }
489 
490 }