001/* 002 * 003 * Licensed to the Apache Software Foundation (ASF) under one 004 * or more contributor license agreements. See the NOTICE file 005 * distributed with this work for additional information 006 * regarding copyright ownership. The ASF licenses this file 007 * to you under the Apache License, Version 2.0 (the 008 * "License"); you may not use this file except in compliance 009 * with the License. You may obtain a copy of the License at 010 * 011 * http://www.apache.org/licenses/LICENSE-2.0 012 * 013 * Unless required by applicable law or agreed to in writing, software 014 * distributed under the License is distributed on an "AS IS" BASIS, 015 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. 016 * See the License for the specific language governing permissions and 017 * limitations under the License. 018 */ 019package org.apache.hadoop.hbase.master; 020import java.io.IOException; 021import java.util.Optional; 022import java.util.concurrent.atomic.AtomicBoolean; 023import org.apache.hadoop.hbase.Server; 024import org.apache.hadoop.hbase.ServerName; 025import org.apache.hadoop.hbase.ZNodeClearer; 026import org.apache.hadoop.hbase.exceptions.DeserializationException; 027import org.apache.hadoop.hbase.monitoring.MonitoredTask; 028import org.apache.hadoop.hbase.zookeeper.MasterAddressTracker; 029import org.apache.hadoop.hbase.zookeeper.ZKListener; 030import org.apache.hadoop.hbase.zookeeper.ZKUtil; 031import org.apache.hadoop.hbase.zookeeper.ZKWatcher; 032import org.apache.hadoop.hbase.zookeeper.ZNodePaths; 033import org.apache.yetus.audience.InterfaceAudience; 034import org.apache.zookeeper.KeeperException; 035import org.slf4j.Logger; 036import org.slf4j.LoggerFactory; 037import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil; 038 039/** 040 * Handles everything on master-side related to master election. 041 * 042 * <p>Listens and responds to ZooKeeper notifications on the master znode, 043 * both <code>nodeCreated</code> and <code>nodeDeleted</code>. 044 * 045 * <p>Contains blocking methods which will hold up backup masters, waiting 046 * for the active master to fail. 047 * 048 * <p>This class is instantiated in the HMaster constructor and the method 049 * #blockUntilBecomingActiveMaster() is called to wait until becoming 050 * the active master of the cluster. 051 */ 052@InterfaceAudience.Private 053public class ActiveMasterManager extends ZKListener { 054 private static final Logger LOG = LoggerFactory.getLogger(ActiveMasterManager.class); 055 056 final AtomicBoolean clusterHasActiveMaster = new AtomicBoolean(false); 057 final AtomicBoolean clusterShutDown = new AtomicBoolean(false); 058 059 // This server's information. Package-private for child implementations. 060 int infoPort; 061 final ServerName sn; 062 final Server master; 063 064 // Active master's server name. Invalidated anytime active master changes (based on ZK 065 // notifications) and lazily fetched on-demand. 066 // ServerName is immutable, so we don't need heavy synchronization around it. 067 volatile ServerName activeMasterServerName; 068 069 /** 070 * @param watcher ZK watcher 071 * @param sn ServerName 072 * @param master In an instance of a Master. 073 */ 074 ActiveMasterManager(ZKWatcher watcher, ServerName sn, Server master) { 075 super(watcher); 076 watcher.registerListener(this); 077 this.sn = sn; 078 this.master = master; 079 } 080 081 // will be set after jetty server is started 082 public void setInfoPort(int infoPort) { 083 this.infoPort = infoPort; 084 } 085 086 @Override 087 public void nodeCreated(String path) { 088 handle(path); 089 } 090 091 @Override 092 public void nodeDeleted(String path) { 093 094 // We need to keep track of the cluster's shutdown status while 095 // we wait on the current master. We consider that, if the cluster 096 // was already in a "shutdown" state when we started, that this master 097 // is part of a new cluster that was started shortly after the old cluster 098 // shut down, so that state is now irrelevant. This means that the shutdown 099 // state must be set while we wait on the active master in order 100 // to shutdown this master. See HBASE-8519. 101 if(path.equals(watcher.getZNodePaths().clusterStateZNode) && !master.isStopped()) { 102 clusterShutDown.set(true); 103 } 104 105 handle(path); 106 } 107 108 void handle(final String path) { 109 if (path.equals(watcher.getZNodePaths().masterAddressZNode) && !master.isStopped()) { 110 handleMasterNodeChange(); 111 } 112 } 113 114 /** 115 * Fetches the active master's ServerName from zookeeper. 116 */ 117 private void fetchAndSetActiveMasterServerName() { 118 LOG.debug("Attempting to fetch active master sn from zk"); 119 try { 120 activeMasterServerName = MasterAddressTracker.getMasterAddress(watcher); 121 } catch (IOException | KeeperException e) { 122 // Log and ignore for now and re-fetch later if needed. 123 LOG.error("Error fetching active master information", e); 124 } 125 } 126 127 public Optional<ServerName> getActiveMasterServerName() { 128 if (!clusterHasActiveMaster.get()) { 129 return Optional.empty(); 130 } 131 if (activeMasterServerName == null) { 132 fetchAndSetActiveMasterServerName(); 133 } 134 // It could still be null, but return whatever we have. 135 return Optional.ofNullable(activeMasterServerName); 136 } 137 138 /** 139 * Handle a change in the master node. Doesn't matter whether this was called 140 * from a nodeCreated or nodeDeleted event because there are no guarantees 141 * that the current state of the master node matches the event at the time of 142 * our next ZK request. 143 * 144 * <p>Uses the watchAndCheckExists method which watches the master address node 145 * regardless of whether it exists or not. If it does exist (there is an 146 * active master), it returns true. Otherwise it returns false. 147 * 148 * <p>A watcher is set which guarantees that this method will get called again if 149 * there is another change in the master node. 150 */ 151 private void handleMasterNodeChange() { 152 // Watch the node and check if it exists. 153 try { 154 synchronized(clusterHasActiveMaster) { 155 if (ZKUtil.watchAndCheckExists(watcher, watcher.getZNodePaths().masterAddressZNode)) { 156 // A master node exists, there is an active master 157 LOG.trace("A master is now available"); 158 clusterHasActiveMaster.set(true); 159 } else { 160 // Node is no longer there, cluster does not have an active master 161 LOG.debug("No master available. Notifying waiting threads"); 162 clusterHasActiveMaster.set(false); 163 // Notify any thread waiting to become the active master 164 clusterHasActiveMaster.notifyAll(); 165 } 166 // Reset the active master sn. Will be re-fetched later if needed. 167 // We don't want to make a synchronous RPC under a monitor. 168 activeMasterServerName = null; 169 } 170 } catch (KeeperException ke) { 171 master.abort("Received an unexpected KeeperException, aborting", ke); 172 } 173 } 174 175 /** 176 * Block until becoming the active master. 177 * 178 * Method blocks until there is not another active master and our attempt 179 * to become the new active master is successful. 180 * 181 * This also makes sure that we are watching the master znode so will be 182 * notified if another master dies. 183 * @param checkInterval the interval to check if the master is stopped 184 * @param startupStatus the monitor status to track the progress 185 * @return True if no issue becoming active master else false if another 186 * master was running or if some other problem (zookeeper, stop flag has been 187 * set on this Master) 188 */ 189 boolean blockUntilBecomingActiveMaster( 190 int checkInterval, MonitoredTask startupStatus) { 191 String backupZNode = ZNodePaths.joinZNode( 192 this.watcher.getZNodePaths().backupMasterAddressesZNode, this.sn.toString()); 193 while (!(master.isAborted() || master.isStopped())) { 194 startupStatus.setStatus("Trying to register in ZK as active master"); 195 // Try to become the active master, watch if there is another master. 196 // Write out our ServerName as versioned bytes. 197 try { 198 if (MasterAddressTracker.setMasterAddress(this.watcher, 199 this.watcher.getZNodePaths().masterAddressZNode, this.sn, infoPort)) { 200 201 // If we were a backup master before, delete our ZNode from the backup 202 // master directory since we are the active now) 203 if (ZKUtil.checkExists(this.watcher, backupZNode) != -1) { 204 LOG.info("Deleting ZNode for " + backupZNode + " from backup master directory"); 205 ZKUtil.deleteNodeFailSilent(this.watcher, backupZNode); 206 } 207 // Save the znode in a file, this will allow to check if we crash in the launch scripts 208 ZNodeClearer.writeMyEphemeralNodeOnDisk(this.sn.toString()); 209 210 // We are the master, return 211 startupStatus.setStatus("Successfully registered as active master."); 212 this.clusterHasActiveMaster.set(true); 213 activeMasterServerName = sn; 214 LOG.info("Registered as active master=" + this.sn); 215 return true; 216 } 217 218 // Invalidate the active master name so that subsequent requests do not get any stale 219 // master information. Will be re-fetched if needed. 220 activeMasterServerName = null; 221 // There is another active master running elsewhere or this is a restart 222 // and the master ephemeral node has not expired yet. 223 this.clusterHasActiveMaster.set(true); 224 225 String msg; 226 byte[] bytes = 227 ZKUtil.getDataAndWatch(this.watcher, this.watcher.getZNodePaths().masterAddressZNode); 228 if (bytes == null) { 229 msg = ("A master was detected, but went down before its address " + 230 "could be read. Attempting to become the next active master"); 231 } else { 232 ServerName currentMaster; 233 try { 234 currentMaster = ProtobufUtil.parseServerNameFrom(bytes); 235 } catch (DeserializationException e) { 236 LOG.warn("Failed parse", e); 237 // Hopefully next time around we won't fail the parse. Dangerous. 238 continue; 239 } 240 if (ServerName.isSameAddress(currentMaster, this.sn)) { 241 msg = ("Current master has this master's address, " + 242 currentMaster + "; master was restarted? Deleting node."); 243 // Hurry along the expiration of the znode. 244 ZKUtil.deleteNode(this.watcher, this.watcher.getZNodePaths().masterAddressZNode); 245 246 // We may have failed to delete the znode at the previous step, but 247 // we delete the file anyway: a second attempt to delete the znode is likely to fail 248 // again. 249 ZNodeClearer.deleteMyEphemeralNodeOnDisk(); 250 } else { 251 msg = "Another master is the active master, " + currentMaster + 252 "; waiting to become the next active master"; 253 } 254 } 255 LOG.info(msg); 256 startupStatus.setStatus(msg); 257 } catch (KeeperException ke) { 258 master.abort("Received an unexpected KeeperException, aborting", ke); 259 return false; 260 } 261 synchronized (this.clusterHasActiveMaster) { 262 while (clusterHasActiveMaster.get() && !master.isStopped()) { 263 try { 264 clusterHasActiveMaster.wait(checkInterval); 265 } catch (InterruptedException e) { 266 // We expect to be interrupted when a master dies, 267 // will fall out if so 268 LOG.debug("Interrupted waiting for master to die", e); 269 } 270 } 271 if (clusterShutDown.get()) { 272 this.master.stop( 273 "Cluster went down before this master became active"); 274 } 275 } 276 } 277 return false; 278 } 279 280 /** 281 * @return True if cluster has an active master. 282 */ 283 boolean hasActiveMaster() { 284 try { 285 if (ZKUtil.checkExists(watcher, watcher.getZNodePaths().masterAddressZNode) >= 0) { 286 return true; 287 } 288 } 289 catch (KeeperException ke) { 290 LOG.info("Received an unexpected KeeperException when checking " + 291 "isActiveMaster : "+ ke); 292 } 293 return false; 294 } 295 296 public void stop() { 297 try { 298 synchronized (clusterHasActiveMaster) { 299 // Master is already stopped, wake up the manager 300 // thread so that it can shutdown soon. 301 clusterHasActiveMaster.notifyAll(); 302 } 303 // If our address is in ZK, delete it on our way out 304 ServerName activeMaster = null; 305 try { 306 activeMaster = MasterAddressTracker.getMasterAddress(this.watcher); 307 } catch (IOException e) { 308 LOG.warn("Failed get of master address: " + e.toString()); 309 } 310 if (activeMaster != null && activeMaster.equals(this.sn)) { 311 ZKUtil.deleteNode(watcher, watcher.getZNodePaths().masterAddressZNode); 312 // We may have failed to delete the znode at the previous step, but 313 // we delete the file anyway: a second attempt to delete the znode is likely to fail again. 314 ZNodeClearer.deleteMyEphemeralNodeOnDisk(); 315 } 316 } catch (KeeperException e) { 317 LOG.debug(this.watcher.prefix("Failed delete of our master address node; " + 318 e.getMessage())); 319 } 320 } 321}