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.HashMap;
25  import java.util.List;
26  import java.util.Map;
27  import java.util.concurrent.CopyOnWriteArrayList;
28  import java.util.concurrent.CountDownLatch;
29  import java.util.regex.Matcher;
30  import java.util.regex.Pattern;
31  
32  import org.apache.commons.logging.Log;
33  import org.apache.commons.logging.LogFactory;
34  import org.apache.hadoop.hbase.classification.InterfaceAudience;
35  import org.apache.hadoop.hbase.security.Superusers;
36  import org.apache.hadoop.conf.Configuration;
37  import org.apache.hadoop.hbase.Abortable;
38  import org.apache.hadoop.hbase.AuthUtil;
39  import org.apache.hadoop.hbase.HConstants;
40  import org.apache.hadoop.hbase.HRegionInfo;
41  import org.apache.hadoop.hbase.ZooKeeperConnectionException;
42  import org.apache.hadoop.security.UserGroupInformation;
43  import org.apache.zookeeper.KeeperException;
44  import org.apache.zookeeper.WatchedEvent;
45  import org.apache.zookeeper.Watcher;
46  import org.apache.zookeeper.ZooDefs;
47  import org.apache.zookeeper.ZooDefs.Ids;
48  import org.apache.zookeeper.ZooDefs.Perms;
49  import org.apache.zookeeper.data.ACL;
50  import org.apache.zookeeper.data.Id;
51  import org.apache.zookeeper.data.Stat;
52  
53  /**
54   * Acts as the single ZooKeeper Watcher.  One instance of this is instantiated
55   * for each Master, RegionServer, and client process.
56   *
57   * <p>This is the only class that implements {@link Watcher}.  Other internal
58   * classes which need to be notified of ZooKeeper events must register with
59   * the local instance of this watcher via {@link #registerListener}.
60   *
61   * <p>This class also holds and manages the connection to ZooKeeper.  Code to
62   * deal with connection related events and exceptions are handled here.
63   */
64  @InterfaceAudience.Private
65  public class ZooKeeperWatcher implements Watcher, Abortable, Closeable {
66    private static final Log LOG = LogFactory.getLog(ZooKeeperWatcher.class);
67  
68    // Identifier for this watcher (for logging only).  It is made of the prefix
69    // passed on construction and the zookeeper sessionid.
70    private String prefix;
71    private String identifier;
72  
73    // zookeeper quorum
74    private String quorum;
75  
76    // zookeeper connection
77    private RecoverableZooKeeper recoverableZooKeeper;
78  
79    // abortable in case of zk failure
80    protected Abortable abortable;
81    // Used if abortable is null
82    private boolean aborted = false;
83  
84    // listeners to be notified
85    private final List<ZooKeeperListener> listeners =
86      new CopyOnWriteArrayList<ZooKeeperListener>();
87  
88    // Used by ZKUtil:waitForZKConnectionIfAuthenticating to wait for SASL
89    // negotiation to complete
90    public CountDownLatch saslLatch = new CountDownLatch(1);
91  
92    // node names
93  
94    // base znode for this cluster
95    public String baseZNode;
96    //znodes containing the locations of the servers hosting the meta replicas
97    private Map<Integer,String> metaReplicaZnodes = new HashMap<Integer, String>();
98    // znode containing ephemeral nodes of the regionservers
99    public String rsZNode;
100   // znode containing ephemeral nodes of the draining regionservers
101   public String drainingZNode;
102   // znode of currently active master
103   private String masterAddressZNode;
104   // znode of this master in backup master directory, if not the active master
105   public String backupMasterAddressesZNode;
106   // znode containing the current cluster state
107   public String clusterStateZNode;
108   // znode used for region transitioning and assignment
109   public String assignmentZNode;
110   // znode used for table disabling/enabling
111   public String tableZNode;
112   // znode containing the unique cluster ID
113   public String clusterIdZNode;
114   // znode used for log splitting work assignment
115   public String splitLogZNode;
116   // znode containing the state of the load balancer
117   public String balancerZNode;
118   // znode containing the lock for the tables
119   public String tableLockZNode;
120   // znode containing the state of recovering regions
121   public String recoveringRegionsZNode;
122   // znode containing namespace descriptors
123   public static String namespaceZNode = "namespace";
124 
125   // Certain ZooKeeper nodes need to be world-readable
126   public static final ArrayList<ACL> CREATOR_ALL_AND_WORLD_READABLE =
127     new ArrayList<ACL>() { {
128       add(new ACL(ZooDefs.Perms.READ,ZooDefs.Ids.ANYONE_ID_UNSAFE));
129       add(new ACL(ZooDefs.Perms.ALL,ZooDefs.Ids.AUTH_IDS));
130     }};
131 
132   public final static String META_ZNODE_PREFIX = "meta-region-server";
133 
134   private final Configuration conf;
135 
136   private final Exception constructorCaller;
137 
138   /* A pattern that matches a Kerberos name, borrowed from Hadoop's KerberosName */
139   private static final Pattern NAME_PATTERN = Pattern.compile("([^/@]*)(/([^/@]*))?@([^/@]*)");
140 
141   /**
142    * Instantiate a ZooKeeper connection and watcher.
143    * @param identifier string that is passed to RecoverableZookeeper to be used as
144    * identifier for this instance. Use null for default.
145    * @throws IOException
146    * @throws ZooKeeperConnectionException
147    */
148   public ZooKeeperWatcher(Configuration conf, String identifier,
149       Abortable abortable) throws ZooKeeperConnectionException, IOException {
150     this(conf, identifier, abortable, false);
151   }
152 
153   /**
154    * Instantiate a ZooKeeper connection and watcher.
155    * @param conf
156    * @param identifier string that is passed to RecoverableZookeeper to be used as identifier for
157    *          this instance. Use null for default.
158    * @param abortable Can be null if there is on error there is no host to abort: e.g. client
159    *          context.
160    * @param canCreateBaseZNode
161    * @throws IOException
162    * @throws ZooKeeperConnectionException
163    */
164   public ZooKeeperWatcher(Configuration conf, String identifier,
165       Abortable abortable, boolean canCreateBaseZNode)
166   throws IOException, ZooKeeperConnectionException {
167     this.conf = conf;
168     // Capture a stack trace now.  Will print it out later if problem so we can
169     // distingush amongst the myriad ZKWs.
170     try {
171       throw new Exception("ZKW CONSTRUCTOR STACK TRACE FOR DEBUGGING");
172     } catch (Exception e) {
173       this.constructorCaller = e;
174     }
175     this.quorum = ZKConfig.getZKQuorumServersString(conf);
176     this.prefix = identifier;
177     // Identifier will get the sessionid appended later below down when we
178     // handle the syncconnect event.
179     this.identifier = identifier + "0x0";
180     this.abortable = abortable;
181     setNodeNames(conf);
182     this.recoverableZooKeeper = ZKUtil.connect(conf, quorum, this, identifier);
183     if (canCreateBaseZNode) {
184       createBaseZNodes();
185     }
186   }
187 
188   private void createBaseZNodes() throws ZooKeeperConnectionException {
189     try {
190       // Create all the necessary "directories" of znodes
191       ZKUtil.createWithParents(this, baseZNode);
192       if (conf.getBoolean("hbase.assignment.usezk", true)) {
193         ZKUtil.createAndFailSilent(this, assignmentZNode);
194       }
195       ZKUtil.createAndFailSilent(this, rsZNode);
196       ZKUtil.createAndFailSilent(this, drainingZNode);
197       ZKUtil.createAndFailSilent(this, tableZNode);
198       ZKUtil.createAndFailSilent(this, splitLogZNode);
199       ZKUtil.createAndFailSilent(this, backupMasterAddressesZNode);
200       ZKUtil.createAndFailSilent(this, tableLockZNode);
201       ZKUtil.createAndFailSilent(this, recoveringRegionsZNode);
202     } catch (KeeperException e) {
203       throw new ZooKeeperConnectionException(
204           prefix("Unexpected KeeperException creating base node"), e);
205     }
206   }
207 
208   /** Returns whether the znode is supposed to be readable by the client
209    * and DOES NOT contain sensitive information (world readable).*/
210   public boolean isClientReadable(String node) {
211     // Developer notice: These znodes are world readable. DO NOT add more znodes here UNLESS
212     // all clients need to access this data to work. Using zk for sharing data to clients (other
213     // than service lookup case is not a recommended design pattern.
214     return
215         node.equals(baseZNode) ||
216         isAnyMetaReplicaZnode(node) ||
217         node.equals(getMasterAddressZNode()) ||
218         node.equals(clusterIdZNode)||
219         node.equals(rsZNode) ||
220         // /hbase/table and /hbase/table/foo is allowed, /hbase/table-lock is not
221         node.equals(tableZNode) ||
222         node.startsWith(tableZNode + "/");
223   }
224 
225   /**
226    * On master start, we check the znode ACLs under the root directory and set the ACLs properly
227    * if needed. If the cluster goes from an unsecure setup to a secure setup, this step is needed
228    * so that the existing znodes created with open permissions are now changed with restrictive
229    * perms.
230    */
231   public void checkAndSetZNodeAcls() {
232     if (!ZKUtil.isSecureZooKeeper(getConfiguration())) {
233       LOG.info("not a secure deployment, proceeding");
234       return;
235     }
236 
237     // Check the base znodes permission first. Only do the recursion if base znode's perms are not
238     // correct.
239     try {
240       List<ACL> actualAcls = recoverableZooKeeper.getAcl(baseZNode, new Stat());
241 
242       if (!isBaseZnodeAclSetup(actualAcls)) {
243         LOG.info("setting znode ACLs");
244         setZnodeAclsRecursive(baseZNode);
245       }
246     } catch(KeeperException.NoNodeException nne) {
247       return;
248     } catch(InterruptedException ie) {
249       interruptedException(ie);
250     } catch (IOException|KeeperException e) {
251       LOG.warn("Received exception while checking and setting zookeeper ACLs", e);
252     }
253   }
254 
255   /**
256    * Set the znode perms recursively. This will do post-order recursion, so that baseZnode ACLs
257    * will be set last in case the master fails in between.
258    * @param znode
259    */
260   private void setZnodeAclsRecursive(String znode) throws KeeperException, InterruptedException {
261     List<String> children = recoverableZooKeeper.getChildren(znode, false);
262 
263     for (String child : children) {
264       setZnodeAclsRecursive(ZKUtil.joinZNode(znode, child));
265     }
266     List<ACL> acls = ZKUtil.createACL(this, znode, true);
267     LOG.info("Setting ACLs for znode:" + znode + " , acl:" + acls);
268     recoverableZooKeeper.setAcl(znode, acls, -1);
269   }
270 
271   /**
272    * Checks whether the ACLs returned from the base znode (/hbase) is set for secure setup.
273    * @param acls acls from zookeeper
274    * @return whether ACLs are set for the base znode
275    * @throws IOException
276    */
277   private boolean isBaseZnodeAclSetup(List<ACL> acls) throws IOException {
278     if (LOG.isDebugEnabled()) {
279       LOG.debug("Checking znode ACLs");
280     }
281     String[] superUsers = conf.getStrings(Superusers.SUPERUSER_CONF_KEY);
282     // Check whether ACL set for all superusers
283     if (superUsers != null && !checkACLForSuperUsers(superUsers, acls)) {
284       return false;
285     }
286 
287     // this assumes that current authenticated user is the same as zookeeper client user
288     // configured via JAAS
289     String hbaseUser = UserGroupInformation.getCurrentUser().getShortUserName();
290 
291     if (acls.isEmpty()) {
292       if (LOG.isDebugEnabled()) {
293         LOG.debug("ACL is empty");
294       }
295       return false;
296     }
297 
298     for (ACL acl : acls) {
299       int perms = acl.getPerms();
300       Id id = acl.getId();
301       // We should only set at most 3 possible ACLs for 3 Ids. One for everyone, one for superuser
302       // and one for the hbase user
303       if (Ids.ANYONE_ID_UNSAFE.equals(id)) {
304         if (perms != Perms.READ) {
305           if (LOG.isDebugEnabled()) {
306             LOG.debug(String.format("permissions for '%s' are not correct: have 0x%x, want 0x%x",
307               id, perms, Perms.READ));
308           }
309           return false;
310         }
311       } else if ("sasl".equals(id.getScheme())) {
312         String name = id.getId();
313         // If ZooKeeper recorded the Kerberos full name in the ACL, use only the shortname
314         Matcher match = NAME_PATTERN.matcher(name);
315         if (match.matches()) {
316           name = match.group(1);
317         }
318         if (name.equals(hbaseUser)) {
319           if (perms != Perms.ALL) {
320             if (LOG.isDebugEnabled()) {
321               LOG.debug(String.format("permissions for '%s' are not correct: have 0x%x, want 0x%x",
322                 id, perms, Perms.ALL));
323             }
324             return false;
325           }
326         } else {
327           if (LOG.isDebugEnabled()) {
328             LOG.debug("Unexpected shortname in SASL ACL: " + id);
329           }
330           return false;
331         }
332       } else {
333         if (LOG.isDebugEnabled()) {
334           LOG.debug("unexpected ACL id '" + id + "'");
335         }
336         return false;
337       }
338     }
339     return true;
340   }
341 
342   /*
343    * Validate whether ACL set for all superusers.
344    */
345   private boolean checkACLForSuperUsers(String[] superUsers, List<ACL> acls) {
346     for (String user : superUsers) {
347       boolean hasAccess = false;
348       // TODO: Validate super group members also when ZK supports setting node ACL for groups.
349       if (!user.startsWith(AuthUtil.GROUP_PREFIX)) {
350         for (ACL acl : acls) {
351           if (user.equals(acl.getId().getId())) {
352             if (acl.getPerms() == Perms.ALL) {
353               hasAccess = true;
354             } else {
355               if (LOG.isDebugEnabled()) {
356                 LOG.debug(String.format(
357                   "superuser '%s' does not have correct permissions: have 0x%x, want 0x%x",
358                   acl.getId().getId(), acl.getPerms(), Perms.ALL));
359               }
360             }
361             break;
362           }
363         }
364         if (!hasAccess) {
365           return false;
366         }
367       }
368     }
369     return true;
370   }
371 
372   @Override
373   public String toString() {
374     return this.identifier + ", quorum=" + quorum + ", baseZNode=" + baseZNode;
375   }
376 
377   /**
378    * Adds this instance's identifier as a prefix to the passed <code>str</code>
379    * @param str String to amend.
380    * @return A new string with this instance's identifier as prefix: e.g.
381    * if passed 'hello world', the returned string could be
382    */
383   public String prefix(final String str) {
384     return this.toString() + " " + str;
385   }
386 
387   /**
388    * Set the local variable node names using the specified configuration.
389    */
390   private void setNodeNames(Configuration conf) {
391     baseZNode = conf.get(HConstants.ZOOKEEPER_ZNODE_PARENT,
392         HConstants.DEFAULT_ZOOKEEPER_ZNODE_PARENT);
393     metaReplicaZnodes.put(0, ZKUtil.joinZNode(baseZNode,
394            conf.get("zookeeper.znode.metaserver", "meta-region-server")));
395     int numMetaReplicas = conf.getInt(HConstants.META_REPLICAS_NUM,
396             HConstants.DEFAULT_META_REPLICA_NUM);
397     for (int i = 1; i < numMetaReplicas; i++) {
398       String str = ZKUtil.joinZNode(baseZNode,
399         conf.get("zookeeper.znode.metaserver", "meta-region-server") + "-" + i);
400       metaReplicaZnodes.put(i, str);
401     }
402     rsZNode = ZKUtil.joinZNode(baseZNode,
403         conf.get("zookeeper.znode.rs", "rs"));
404     drainingZNode = ZKUtil.joinZNode(baseZNode,
405         conf.get("zookeeper.znode.draining.rs", "draining"));
406     masterAddressZNode = ZKUtil.joinZNode(baseZNode,
407         conf.get("zookeeper.znode.master", "master"));
408     backupMasterAddressesZNode = ZKUtil.joinZNode(baseZNode,
409         conf.get("zookeeper.znode.backup.masters", "backup-masters"));
410     clusterStateZNode = ZKUtil.joinZNode(baseZNode,
411         conf.get("zookeeper.znode.state", "running"));
412     assignmentZNode = ZKUtil.joinZNode(baseZNode,
413         conf.get("zookeeper.znode.unassigned", "region-in-transition"));
414     tableZNode = ZKUtil.joinZNode(baseZNode,
415         conf.get("zookeeper.znode.tableEnableDisable", "table"));
416     clusterIdZNode = ZKUtil.joinZNode(baseZNode,
417         conf.get("zookeeper.znode.clusterId", "hbaseid"));
418     splitLogZNode = ZKUtil.joinZNode(baseZNode,
419         conf.get("zookeeper.znode.splitlog", HConstants.SPLIT_LOGDIR_NAME));
420     balancerZNode = ZKUtil.joinZNode(baseZNode,
421         conf.get("zookeeper.znode.balancer", "balancer"));
422     tableLockZNode = ZKUtil.joinZNode(baseZNode,
423         conf.get("zookeeper.znode.tableLock", "table-lock"));
424     recoveringRegionsZNode = ZKUtil.joinZNode(baseZNode,
425         conf.get("zookeeper.znode.recovering.regions", "recovering-regions"));
426     namespaceZNode = ZKUtil.joinZNode(baseZNode,
427         conf.get("zookeeper.znode.namespace", "namespace"));
428   }
429 
430   /**
431    * Is the znode of any meta replica
432    * @param node
433    * @return true or false
434    */
435   public boolean isAnyMetaReplicaZnode(String node) {
436     if (metaReplicaZnodes.values().contains(node)) {
437       return true;
438     }
439     return false;
440   }
441 
442   /**
443    * Is it the default meta replica's znode
444    * @param node
445    * @return true or false
446    */
447   public boolean isDefaultMetaReplicaZnode(String node) {
448     if (getZNodeForReplica(HRegionInfo.DEFAULT_REPLICA_ID).equals(node)) {
449       return true;
450     }
451     return false;
452   }
453 
454   /**
455    * Get the znodes corresponding to the meta replicas from ZK
456    * @return list of znodes
457    * @throws KeeperException
458    */
459   public List<String> getMetaReplicaNodes() throws KeeperException {
460     List<String> childrenOfBaseNode = ZKUtil.listChildrenNoWatch(this, baseZNode);
461     List<String> metaReplicaNodes = new ArrayList<String>(2);
462     String pattern = conf.get("zookeeper.znode.metaserver","meta-region-server");
463     for (String child : childrenOfBaseNode) {
464       if (child.startsWith(pattern)) metaReplicaNodes.add(child);
465     }
466     return metaReplicaNodes;
467   }
468 
469   /**
470    * Get the znode string corresponding to a replicaId
471    * @param replicaId
472    * @return znode
473    */
474   public String getZNodeForReplica(int replicaId) {
475     String str = metaReplicaZnodes.get(replicaId);
476     // return a newly created path but don't update the cache of paths
477     // This is mostly needed for tests that attempt to create meta replicas
478     // from outside the master
479     if (str == null) {
480       str = ZKUtil.joinZNode(baseZNode,
481           conf.get("zookeeper.znode.metaserver", "meta-region-server") + "-" + replicaId);
482     }
483     return str;
484   }
485 
486   /**
487    * Parse the meta replicaId from the passed znode
488    * @param znode
489    * @return replicaId
490    */
491   public int getMetaReplicaIdFromZnode(String znode) {
492     String pattern = conf.get("zookeeper.znode.metaserver","meta-region-server");
493     if (znode.equals(pattern)) return HRegionInfo.DEFAULT_REPLICA_ID;
494     // the non-default replicas are of the pattern meta-region-server-<replicaId>
495     String nonDefaultPattern = pattern + "-";
496     return Integer.parseInt(znode.substring(nonDefaultPattern.length()));
497   }
498 
499   /**
500    * Register the specified listener to receive ZooKeeper events.
501    * @param listener
502    */
503   public void registerListener(ZooKeeperListener listener) {
504     listeners.add(listener);
505   }
506 
507   /**
508    * Register the specified listener to receive ZooKeeper events and add it as
509    * the first in the list of current listeners.
510    * @param listener
511    */
512   public void registerListenerFirst(ZooKeeperListener listener) {
513     listeners.add(0, listener);
514   }
515 
516   public void unregisterListener(ZooKeeperListener listener) {
517     listeners.remove(listener);
518   }
519 
520   /**
521    * Clean all existing listeners
522    */
523   public void unregisterAllListeners() {
524     listeners.clear();
525   }
526 
527   /**
528    * Get a copy of current registered listeners
529    */
530   public List<ZooKeeperListener> getListeners() {
531     return new ArrayList<ZooKeeperListener>(listeners);
532   }
533 
534   /**
535    * @return The number of currently registered listeners
536    */
537   public int getNumberOfListeners() {
538     return listeners.size();
539   }
540 
541   /**
542    * Get the connection to ZooKeeper.
543    * @return connection reference to zookeeper
544    */
545   public RecoverableZooKeeper getRecoverableZooKeeper() {
546     return recoverableZooKeeper;
547   }
548 
549   public void reconnectAfterExpiration() throws IOException, KeeperException, InterruptedException {
550     recoverableZooKeeper.reconnectAfterExpiration();
551   }
552 
553   /**
554    * Get the quorum address of this instance.
555    * @return quorum string of this zookeeper connection instance
556    */
557   public String getQuorum() {
558     return quorum;
559   }
560 
561   /**
562    * @return the base znode of this zookeeper connection instance.
563    */
564   public String getBaseZNode() {
565     return baseZNode;
566   }
567 
568   /**
569    * Method called from ZooKeeper for events and connection status.
570    * <p>
571    * Valid events are passed along to listeners.  Connection status changes
572    * are dealt with locally.
573    */
574   @Override
575   public void process(WatchedEvent event) {
576     LOG.debug(prefix("Received ZooKeeper Event, " +
577         "type=" + event.getType() + ", " +
578         "state=" + event.getState() + ", " +
579         "path=" + event.getPath()));
580 
581     switch(event.getType()) {
582 
583       // If event type is NONE, this is a connection status change
584       case None: {
585         connectionEvent(event);
586         break;
587       }
588 
589       // Otherwise pass along to the listeners
590 
591       case NodeCreated: {
592         for(ZooKeeperListener listener : listeners) {
593           listener.nodeCreated(event.getPath());
594         }
595         break;
596       }
597 
598       case NodeDeleted: {
599         for(ZooKeeperListener listener : listeners) {
600           listener.nodeDeleted(event.getPath());
601         }
602         break;
603       }
604 
605       case NodeDataChanged: {
606         for(ZooKeeperListener listener : listeners) {
607           listener.nodeDataChanged(event.getPath());
608         }
609         break;
610       }
611 
612       case NodeChildrenChanged: {
613         for(ZooKeeperListener listener : listeners) {
614           listener.nodeChildrenChanged(event.getPath());
615         }
616         break;
617       }
618     }
619   }
620 
621   // Connection management
622 
623   /**
624    * Called when there is a connection-related event via the Watcher callback.
625    * <p>
626    * If Disconnected or Expired, this should shutdown the cluster. But, since
627    * we send a KeeperException.SessionExpiredException along with the abort
628    * call, it's possible for the Abortable to catch it and try to create a new
629    * session with ZooKeeper. This is what the client does in HCM.
630    * <p>
631    * @param event
632    */
633   private void connectionEvent(WatchedEvent event) {
634     switch(event.getState()) {
635       case SyncConnected:
636         // Now, this callback can be invoked before the this.zookeeper is set.
637         // Wait a little while.
638         long finished = System.currentTimeMillis() +
639           this.conf.getLong("hbase.zookeeper.watcher.sync.connected.wait", 2000);
640         while (System.currentTimeMillis() < finished) {
641           try {
642             Thread.sleep(1);
643           } catch (InterruptedException e) {
644             LOG.warn("Interrupted while sleeping");
645             throw new RuntimeException("Interrupted while waiting for" +
646                 " recoverableZooKeeper is set");
647           }
648           if (this.recoverableZooKeeper != null) break;
649         }
650 
651         if (this.recoverableZooKeeper == null) {
652           LOG.error("ZK is null on connection event -- see stack trace " +
653             "for the stack trace when constructor was called on this zkw",
654             this.constructorCaller);
655           throw new NullPointerException("ZK is null");
656         }
657         this.identifier = this.prefix + "-0x" +
658           Long.toHexString(this.recoverableZooKeeper.getSessionId());
659         // Update our identifier.  Otherwise ignore.
660         LOG.debug(this.identifier + " connected");
661         break;
662 
663       // Abort the server if Disconnected or Expired
664       case Disconnected:
665         LOG.debug(prefix("Received Disconnected from ZooKeeper, ignoring"));
666         break;
667 
668       case Expired:
669         String msg = prefix(this.identifier + " received expired from " +
670           "ZooKeeper, aborting");
671         // TODO: One thought is to add call to ZooKeeperListener so say,
672         // ZooKeeperNodeTracker can zero out its data values.
673         if (this.abortable != null) {
674           this.abortable.abort(msg, new KeeperException.SessionExpiredException());
675         }
676         break;
677 
678       case ConnectedReadOnly:
679       case SaslAuthenticated:
680       case AuthFailed:
681         break;
682 
683       default:
684         throw new IllegalStateException("Received event is not valid: " + event.getState());
685     }
686   }
687 
688   /**
689    * Forces a synchronization of this ZooKeeper client connection.
690    * <p>
691    * Executing this method before running other methods will ensure that the
692    * subsequent operations are up-to-date and consistent as of the time that
693    * the sync is complete.
694    * <p>
695    * This is used for compareAndSwap type operations where we need to read the
696    * data of an existing node and delete or transition that node, utilizing the
697    * previously read version and data.  We want to ensure that the version read
698    * is up-to-date from when we begin the operation.
699    */
700   public void sync(String path) throws KeeperException {
701     this.recoverableZooKeeper.sync(path, null, null);
702   }
703 
704   /**
705    * Handles KeeperExceptions in client calls.
706    * <p>
707    * This may be temporary but for now this gives one place to deal with these.
708    * <p>
709    * TODO: Currently this method rethrows the exception to let the caller handle
710    * <p>
711    * @param ke
712    * @throws KeeperException
713    */
714   public void keeperException(KeeperException ke)
715   throws KeeperException {
716     LOG.error(prefix("Received unexpected KeeperException, re-throwing exception"), ke);
717     throw ke;
718   }
719 
720   /**
721    * Handles InterruptedExceptions in client calls.
722    * <p>
723    * This may be temporary but for now this gives one place to deal with these.
724    * <p>
725    * TODO: Currently, this method does nothing.
726    *       Is this ever expected to happen?  Do we abort or can we let it run?
727    *       Maybe this should be logged as WARN?  It shouldn't happen?
728    * <p>
729    * @param ie
730    */
731   public void interruptedException(InterruptedException ie) {
732     LOG.debug(prefix("Received InterruptedException, doing nothing here"), ie);
733     // At least preserver interrupt.
734     Thread.currentThread().interrupt();
735     // no-op
736   }
737 
738   /**
739    * Close the connection to ZooKeeper.
740    *
741    * @throws InterruptedException
742    */
743   @Override
744   public void close() {
745     try {
746       if (recoverableZooKeeper != null) {
747         recoverableZooKeeper.close();
748       }
749     } catch (InterruptedException e) {
750       Thread.currentThread().interrupt();
751     }
752   }
753 
754   public Configuration getConfiguration() {
755     return conf;
756   }
757 
758   @Override
759   public void abort(String why, Throwable e) {
760     if (this.abortable != null) this.abortable.abort(why, e);
761     else this.aborted = true;
762   }
763 
764   @Override
765   public boolean isAborted() {
766     return this.abortable == null? this.aborted: this.abortable.isAborted();
767   }
768 
769   /**
770    * @return Path to the currently active master.
771    */
772   public String getMasterAddressZNode() {
773     return this.masterAddressZNode;
774   }
775 
776 }