View Javadoc

1   /**
2    * Licensed to the Apache Software Foundation (ASF) under one
3    * or more contributor license agreements.  See the NOTICE file
4    * distributed with this work for additional information
5    * regarding copyright ownership.  The ASF licenses this file
6    * to you under the Apache License, Version 2.0 (the
7    * "License"); you may not use this file except in compliance
8    * with the License.  You may obtain a copy of the License at
9    *
10   *     http://www.apache.org/licenses/LICENSE-2.0
11   *
12   * Unless required by applicable law or agreed to in writing, software
13   * distributed under the License is distributed on an "AS IS" BASIS,
14   * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15   * See the License for the specific language governing permissions and
16   * limitations under the License.
17   */
18  package org.apache.hadoop.hbase.zookeeper;
19  
20  import org.apache.hadoop.hbase.classification.InterfaceAudience;
21  import org.apache.hadoop.hbase.Abortable;
22  import org.apache.hadoop.hbase.HConstants;
23  import org.apache.hadoop.hbase.ServerName;
24  import org.apache.hadoop.hbase.exceptions.DeserializationException;
25  import org.apache.hadoop.hbase.protobuf.ProtobufUtil;
26  import org.apache.hadoop.hbase.protobuf.generated.HBaseProtos;
27  import org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos;
28  import org.apache.zookeeper.KeeperException;
29  import org.apache.zookeeper.data.Stat;
30  
31  import java.io.IOException;
32  import java.io.InterruptedIOException;
33  import com.google.protobuf.InvalidProtocolBufferException;
34  
35  /**
36   * Manages the location of the current active Master for the RegionServer.
37   * <p>
38   * Listens for ZooKeeper events related to the master address. The node
39   * <code>/master</code> will contain the address of the current master.
40   * This listener is interested in
41   * <code>NodeDeleted</code> and <code>NodeCreated</code> events on
42   * <code>/master</code>.
43   * <p>
44   * Utilizes {@link ZooKeeperNodeTracker} for zk interactions.
45   * <p>
46   * You can get the current master via {@link #getMasterAddress()} or via
47   * {@link #getMasterAddress(ZooKeeperWatcher)} if you do not have a running
48   * instance of this Tracker in your context.
49   * <p>
50   * This class also includes utility for interacting with the master znode, for
51   * writing and reading the znode content.
52   */
53  @InterfaceAudience.Private
54  public class MasterAddressTracker extends ZooKeeperNodeTracker {
55    /**
56     * Construct a master address listener with the specified
57     * <code>zookeeper</code> reference.
58     * <p>
59     * This constructor does not trigger any actions, you must call methods
60     * explicitly.  Normally you will just want to execute {@link #start()} to
61     * begin tracking of the master address.
62     *
63     * @param watcher zk reference and watcher
64     * @param abortable abortable in case of fatal error
65     */
66    public MasterAddressTracker(ZooKeeperWatcher watcher, Abortable abortable) {
67      super(watcher, watcher.getMasterAddressZNode(), abortable);
68    }
69  
70    /**
71     * Get the address of the current master if one is available.  Returns null
72     * if no current master.
73     * @return Server name or null if timed out.
74     */
75    public ServerName getMasterAddress() {
76      return getMasterAddress(false);
77    }
78  
79    /**
80     * Get the info port of the current master of one is available.
81     * Return 0 if no current master or zookeeper is unavailable
82     * @return info port or 0 if timed out
83     */
84    public int getMasterInfoPort() {
85      try {
86        final ZooKeeperProtos.Master master = parse(this.getData(false));
87        if (master == null) {
88          return 0;
89        }
90        return master.getInfoPort();
91      } catch (DeserializationException e) {
92        LOG.warn("Failed parse master zk node data", e);
93        return 0;
94      }
95    }
96    /**
97     * Get the info port of the backup master if it is available.
98     * Return 0 if no backup master or zookeeper is unavailable
99     * @param sn server name of backup master
100    * @return info port or 0 if timed out or exceptions
101    */
102   public int getBackupMasterInfoPort(final ServerName sn) {
103     String backupZNode = ZKUtil.joinZNode(watcher.backupMasterAddressesZNode, sn.toString());
104     try {
105       byte[] data = ZKUtil.getData(watcher, backupZNode);
106       final ZooKeeperProtos.Master backup = parse(data);
107       if (backup == null) {
108         return 0;
109       }
110       return backup.getInfoPort();
111     } catch (Exception e) {
112       LOG.warn("Failed to get backup master: " + sn + "'s info port.", e);
113       return 0;
114     }
115   }
116 
117   /**
118    * Get the address of the current master if one is available.  Returns null
119    * if no current master. If refresh is set, try to load the data from ZK again,
120    * otherwise, cached data will be used.
121    *
122    * @param refresh whether to refresh the data by calling ZK directly.
123    * @return Server name or null if timed out.
124    */
125   public ServerName getMasterAddress(final boolean refresh) {
126     try {
127       return ServerName.parseFrom(super.getData(refresh));
128     } catch (DeserializationException e) {
129       LOG.warn("Failed parse", e);
130       return null;
131     }
132   }
133 
134   /**
135    * Get master address.
136    * Use this instead of {@link #getMasterAddress()} if you do not have an
137    * instance of this tracker in your context.
138    * @param zkw ZooKeeperWatcher to use
139    * @return ServerName stored in the the master address znode or null if no
140    * znode present.
141    * @throws KeeperException
142    * @throws IOException
143    */
144   public static ServerName getMasterAddress(final ZooKeeperWatcher zkw)
145   throws KeeperException, IOException {
146     byte [] data;
147     try {
148       data = ZKUtil.getData(zkw, zkw.getMasterAddressZNode());
149     } catch (InterruptedException e) {
150       throw new InterruptedIOException();
151     }
152     // TODO javadoc claims we return null in this case. :/
153     if (data == null){
154       throw new IOException("Can't get master address from ZooKeeper; znode data == null");
155     }
156     try {
157       return ServerName.parseFrom(data);
158     } catch (DeserializationException e) {
159       KeeperException ke = new KeeperException.DataInconsistencyException();
160       ke.initCause(e);
161       throw ke;
162     }
163   }
164 
165   /**
166    * Get master info port.
167    * Use this instead of {@link #getMasterInfoPort()} if you do not have an
168    * instance of this tracker in your context.
169    * @param zkw ZooKeeperWatcher to use
170    * @return master info port in the the master address znode or null if no
171    * znode present.
172    * // TODO can't return null for 'int' return type. non-static verison returns 0
173    * @throws KeeperException
174    * @throws IOException
175    */
176   public static int getMasterInfoPort(final ZooKeeperWatcher zkw) throws KeeperException,
177       IOException {
178     byte[] data;
179     try {
180       data = ZKUtil.getData(zkw, zkw.getMasterAddressZNode());
181     } catch (InterruptedException e) {
182       throw new InterruptedIOException();
183     }
184     // TODO javadoc claims we return null in this case. :/
185     if (data == null) {
186       throw new IOException("Can't get master address from ZooKeeper; znode data == null");
187     }
188     try {
189       return parse(data).getInfoPort();
190     } catch (DeserializationException e) {
191       KeeperException ke = new KeeperException.DataInconsistencyException();
192       ke.initCause(e);
193       throw ke;
194     }
195   }
196 
197   /**
198    * Set master address into the <code>master</code> znode or into the backup
199    * subdirectory of backup masters; switch off the passed in <code>znode</code>
200    * path.
201    * @param zkw The ZooKeeperWatcher to use.
202    * @param znode Where to create the znode; could be at the top level or it
203    * could be under backup masters
204    * @param master ServerName of the current master must not be null.
205    * @return true if node created, false if not; a watch is set in both cases
206    * @throws KeeperException
207    */
208   public static boolean setMasterAddress(final ZooKeeperWatcher zkw,
209       final String znode, final ServerName master, int infoPort)
210   throws KeeperException {
211     return ZKUtil.createEphemeralNodeAndWatch(zkw, znode, toByteArray(master, infoPort));
212   }
213 
214   /**
215    * Check if there is a master available.
216    * @return true if there is a master set, false if not.
217    */
218   public boolean hasMaster() {
219     return super.getData(false) != null;
220   }
221 
222   /**
223    * @param sn must not be null
224    * @return Content of the master znode as a serialized pb with the pb
225    * magic as prefix.
226    */
227   static byte[] toByteArray(final ServerName sn, int infoPort) {
228     ZooKeeperProtos.Master.Builder mbuilder = ZooKeeperProtos.Master.newBuilder();
229     HBaseProtos.ServerName.Builder snbuilder = HBaseProtos.ServerName.newBuilder();
230     snbuilder.setHostName(sn.getHostname());
231     snbuilder.setPort(sn.getPort());
232     snbuilder.setStartCode(sn.getStartcode());
233     mbuilder.setMaster(snbuilder.build());
234     mbuilder.setRpcVersion(HConstants.RPC_CURRENT_VERSION);
235     mbuilder.setInfoPort(infoPort);
236     return ProtobufUtil.prependPBMagic(mbuilder.build().toByteArray());
237   }
238 
239   /**
240    * @param data zookeeper data. may be null
241    * @return pb object of master, null if no active master
242    * @throws DeserializationException
243    */
244   public static ZooKeeperProtos.Master parse(byte[] data) throws DeserializationException {
245     if (data == null) {
246       return null;
247     }
248     int prefixLen = ProtobufUtil.lengthOfPBMagic();
249     try {
250       return ZooKeeperProtos.Master.PARSER.parseFrom(data, prefixLen, data.length - prefixLen);
251     } catch (InvalidProtocolBufferException e) {
252       throw new DeserializationException(e);
253     }
254   }
255   /**
256    * delete the master znode if its content is same as the parameter
257    * @param zkw must not be null
258    * @param content must not be null
259    */
260   public static boolean deleteIfEquals(ZooKeeperWatcher zkw, final String content) {
261     if (content == null){
262       throw new IllegalArgumentException("Content must not be null");
263     }
264 
265     try {
266       Stat stat = new Stat();
267       byte[] data = ZKUtil.getDataNoWatch(zkw, zkw.getMasterAddressZNode(), stat);
268       ServerName sn = ServerName.parseFrom(data);
269       if (sn != null && content.equals(sn.toString())) {
270         return (ZKUtil.deleteNode(zkw, zkw.getMasterAddressZNode(), stat.getVersion()));
271       }
272     } catch (KeeperException e) {
273       LOG.warn("Can't get or delete the master znode", e);
274     } catch (DeserializationException e) {
275       LOG.warn("Can't get or delete the master znode", e);
276     }
277 
278     return false;
279   }
280 }