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.master;
20  
21  import java.io.File;
22  import java.io.IOException;
23  import java.util.List;
24  
25  import org.apache.commons.cli.CommandLine;
26  import org.apache.commons.cli.GnuParser;
27  import org.apache.commons.cli.Options;
28  import org.apache.commons.cli.ParseException;
29  import org.apache.commons.logging.Log;
30  import org.apache.commons.logging.LogFactory;
31  import org.apache.hadoop.hbase.classification.InterfaceAudience;
32  import org.apache.hadoop.conf.Configuration;
33  import org.apache.hadoop.hbase.CoordinatedStateManager;
34  import org.apache.hadoop.hbase.CoordinatedStateManagerFactory;
35  import org.apache.hadoop.hbase.HConstants;
36  import org.apache.hadoop.hbase.LocalHBaseCluster;
37  import org.apache.hadoop.hbase.MasterNotRunningException;
38  import org.apache.hadoop.hbase.ZNodeClearer;
39  import org.apache.hadoop.hbase.ZooKeeperConnectionException;
40  import org.apache.hadoop.hbase.client.Admin;
41  import org.apache.hadoop.hbase.client.HBaseAdmin;
42  import org.apache.hadoop.hbase.regionserver.HRegionServer;
43  import org.apache.hadoop.hbase.util.JVMClusterUtil;
44  import org.apache.hadoop.hbase.util.ServerCommandLine;
45  import org.apache.hadoop.hbase.zookeeper.MiniZooKeeperCluster;
46  import org.apache.hadoop.hbase.zookeeper.ZKUtil;
47  import org.apache.hadoop.metrics2.lib.DefaultMetricsSystem;
48  import org.apache.zookeeper.KeeperException;
49  
50  @InterfaceAudience.Private
51  public class HMasterCommandLine extends ServerCommandLine {
52    private static final Log LOG = LogFactory.getLog(HMasterCommandLine.class);
53  
54    private static final String USAGE =
55      "Usage: Master [opts] start|stop|clear\n" +
56      " start  Start Master. If local mode, start Master and RegionServer in same JVM\n" +
57      " stop   Start cluster shutdown; Master signals RegionServer shutdown\n" +
58      " clear  Delete the master znode in ZooKeeper after a master crashes\n "+
59      " where [opts] are:\n" +
60      "   --minRegionServers=<servers>   Minimum RegionServers needed to host user tables.\n" +
61      "   --localRegionServers=<servers> " +
62        "RegionServers to start in master process when in standalone mode.\n" +
63      "   --masters=<servers>            Masters to start in this process.\n" +
64      "   --backup                       Master should start in backup mode";
65  
66    private final Class<? extends HMaster> masterClass;
67  
68    public HMasterCommandLine(Class<? extends HMaster> masterClass) {
69      this.masterClass = masterClass;
70    }
71  
72    protected String getUsage() {
73      return USAGE;
74    }
75  
76  
77    public int run(String args[]) throws Exception {
78      Options opt = new Options();
79      opt.addOption("localRegionServers", true,
80        "RegionServers to start in master process when running standalone");
81      opt.addOption("masters", true, "Masters to start in this process");
82      opt.addOption("minRegionServers", true, "Minimum RegionServers needed to host user tables");
83      opt.addOption("backup", false, "Do not try to become HMaster until the primary fails");
84  
85      CommandLine cmd;
86      try {
87        cmd = new GnuParser().parse(opt, args);
88      } catch (ParseException e) {
89        LOG.error("Could not parse: ", e);
90        usage(null);
91        return 1;
92      }
93  
94  
95      if (cmd.hasOption("minRegionServers")) {
96        String val = cmd.getOptionValue("minRegionServers");
97        getConf().setInt("hbase.regions.server.count.min", Integer.parseInt(val));
98        LOG.debug("minRegionServers set to " + val);
99      }
100 
101     // minRegionServers used to be minServers.  Support it too.
102     if (cmd.hasOption("minServers")) {
103       String val = cmd.getOptionValue("minServers");
104       getConf().setInt("hbase.regions.server.count.min", Integer.parseInt(val));
105       LOG.debug("minServers set to " + val);
106     }
107 
108     // check if we are the backup master - override the conf if so
109     if (cmd.hasOption("backup")) {
110       getConf().setBoolean(HConstants.MASTER_TYPE_BACKUP, true);
111     }
112 
113     // How many regionservers to startup in this process (we run regionservers in same process as
114     // master when we are in local/standalone mode. Useful testing)
115     if (cmd.hasOption("localRegionServers")) {
116       String val = cmd.getOptionValue("localRegionServers");
117       getConf().setInt("hbase.regionservers", Integer.parseInt(val));
118       LOG.debug("localRegionServers set to " + val);
119     }
120     // How many masters to startup inside this process; useful testing
121     if (cmd.hasOption("masters")) {
122       String val = cmd.getOptionValue("masters");
123       getConf().setInt("hbase.masters", Integer.parseInt(val));
124       LOG.debug("masters set to " + val);
125     }
126 
127     @SuppressWarnings("unchecked")
128     List<String> remainingArgs = cmd.getArgList();
129     if (remainingArgs.size() != 1) {
130       usage(null);
131       return 1;
132     }
133 
134     String command = remainingArgs.get(0);
135 
136     if ("start".equals(command)) {
137       return startMaster();
138     } else if ("stop".equals(command)) {
139       return stopMaster();
140     } else if ("clear".equals(command)) {
141       return (ZNodeClearer.clear(getConf()) ? 0 : 1);
142     } else {
143       usage("Invalid command: " + command);
144       return 1;
145     }
146   }
147 
148   private int startMaster() {
149     Configuration conf = getConf();
150     try {
151       // If 'local', defer to LocalHBaseCluster instance.  Starts master
152       // and regionserver both in the one JVM.
153       if (LocalHBaseCluster.isLocal(conf)) {
154         DefaultMetricsSystem.setMiniClusterMode(true);
155         final MiniZooKeeperCluster zooKeeperCluster = new MiniZooKeeperCluster(conf);
156         File zkDataPath = new File(conf.get(HConstants.ZOOKEEPER_DATA_DIR));
157 
158         // find out the default client port
159         int zkClientPort = 0;
160 
161         // If the zookeeper client port is specified in server quorum, use it.
162         String zkserver = conf.get(HConstants.ZOOKEEPER_QUORUM);
163         if (zkserver != null) {
164           String[] zkservers = zkserver.split(",");
165 
166           if (zkservers.length > 1) {
167             // In local mode deployment, we have the master + a region server and zookeeper server
168             // started in the same process. Therefore, we only support one zookeeper server.
169             String errorMsg = "Could not start ZK with " + zkservers.length +
170                 " ZK servers in local mode deployment. Aborting as clients (e.g. shell) will not "
171                 + "be able to find this ZK quorum.";
172               System.err.println(errorMsg);
173               throw new IOException(errorMsg);
174           }
175 
176           String[] parts = zkservers[0].split(":");
177 
178           if (parts.length == 2) {
179             // the second part is the client port
180             zkClientPort = Integer.parseInt(parts [1]);
181           }
182         }
183         // If the client port could not be find in server quorum conf, try another conf
184         if (zkClientPort == 0) {
185           zkClientPort = conf.getInt(HConstants.ZOOKEEPER_CLIENT_PORT, 0);
186           // The client port has to be set by now; if not, throw exception.
187           if (zkClientPort == 0) {
188             throw new IOException("No config value for " + HConstants.ZOOKEEPER_CLIENT_PORT);
189           }
190         }
191         zooKeeperCluster.setDefaultClientPort(zkClientPort);
192         // set the ZK tick time if specified
193         int zkTickTime = conf.getInt(HConstants.ZOOKEEPER_TICK_TIME, 0);
194         if (zkTickTime > 0) {
195           zooKeeperCluster.setTickTime(zkTickTime);
196         }
197 
198         // login the zookeeper server principal (if using security)
199         ZKUtil.loginServer(conf, HConstants.ZK_SERVER_KEYTAB_FILE,
200           HConstants.ZK_SERVER_KERBEROS_PRINCIPAL, null);
201         int localZKClusterSessionTimeout =
202           conf.getInt(HConstants.ZK_SESSION_TIMEOUT + ".localHBaseCluster", 10*1000);
203         conf.setInt(HConstants.ZK_SESSION_TIMEOUT, localZKClusterSessionTimeout);
204         LOG.info("Starting a zookeeper cluster");
205         int clientPort = zooKeeperCluster.startup(zkDataPath);
206         if (clientPort != zkClientPort) {
207           String errorMsg = "Could not start ZK at requested port of " +
208             zkClientPort + ".  ZK was started at port: " + clientPort +
209             ".  Aborting as clients (e.g. shell) will not be able to find " +
210             "this ZK quorum.";
211           System.err.println(errorMsg);
212           throw new IOException(errorMsg);
213         }
214         conf.set(HConstants.ZOOKEEPER_CLIENT_PORT, Integer.toString(clientPort));
215 
216         // Need to have the zk cluster shutdown when master is shutdown.
217         // Run a subclass that does the zk cluster shutdown on its way out.
218         int mastersCount = conf.getInt("hbase.masters", 1);
219         int regionServersCount = conf.getInt("hbase.regionservers", 1);
220         LOG.info("Starting up instance of localHBaseCluster; master=" + mastersCount +
221           ", regionserversCount=" + regionServersCount);
222         LocalHBaseCluster cluster = new LocalHBaseCluster(conf, mastersCount, regionServersCount,
223           LocalHMaster.class, HRegionServer.class);
224         ((LocalHMaster)cluster.getMaster(0)).setZKCluster(zooKeeperCluster);
225         cluster.startup();
226         waitOnMasterThreads(cluster);
227       } else {
228         logProcessInfo(getConf());
229         CoordinatedStateManager csm =
230           CoordinatedStateManagerFactory.getCoordinatedStateManager(conf);
231         HMaster master = HMaster.constructMaster(masterClass, conf, csm);
232         if (master.isStopped()) {
233           LOG.info("Won't bring the Master up as a shutdown is requested");
234           return 1;
235         }
236         master.start();
237         master.join();
238         if(master.isAborted())
239           throw new RuntimeException("HMaster Aborted");
240       }
241     } catch (Throwable t) {
242       LOG.error("Master exiting", t);
243       return 1;
244     }
245     return 0;
246   }
247 
248   @SuppressWarnings("resource")
249   private int stopMaster() {
250     Admin adm = null;
251     try {
252       Configuration conf = getConf();
253       // Don't try more than once
254       conf.setInt(HConstants.HBASE_CLIENT_RETRIES_NUMBER, 1);
255       adm = new HBaseAdmin(getConf());
256       adm.shutdown();
257     } catch (MasterNotRunningException e) {
258       LOG.error("Master not running");
259       return 1;
260     } catch (ZooKeeperConnectionException e) {
261       LOG.error("ZooKeeper not available");
262       return 1;
263     } catch (IOException e) {
264       LOG.error("Got IOException: " +e.getMessage(), e);
265       return 1;
266     } catch (Throwable t) {
267       LOG.error("Failed to stop master", t);
268       return 1;
269     } finally {
270       if (adm != null) {
271         try {
272           adm.close();
273         } catch (Throwable t) {
274           LOG.error("Failed to close Admin", t);
275           return 1;
276         }
277       }
278     }
279     return 0;
280   }
281 
282   private void waitOnMasterThreads(LocalHBaseCluster cluster) throws InterruptedException{
283     List<JVMClusterUtil.MasterThread> masters = cluster.getMasters();
284     List<JVMClusterUtil.RegionServerThread> regionservers = cluster.getRegionServers();
285 
286     if (masters != null) {
287       for (JVMClusterUtil.MasterThread t : masters) {
288         t.join();
289         if(t.getMaster().isAborted()) {
290           closeAllRegionServerThreads(regionservers);
291           throw new RuntimeException("HMaster Aborted");
292         }
293       }
294     }
295   }
296 
297   private static void closeAllRegionServerThreads(
298       List<JVMClusterUtil.RegionServerThread> regionservers) {
299     for(JVMClusterUtil.RegionServerThread t : regionservers){
300       t.getRegionServer().stop("HMaster Aborted; Bringing down regions servers");
301     }
302   }
303 
304   /*
305    * Version of master that will shutdown the passed zk cluster on its way out.
306    */
307   public static class LocalHMaster extends HMaster {
308     private MiniZooKeeperCluster zkcluster = null;
309 
310     public LocalHMaster(Configuration conf, CoordinatedStateManager csm)
311     throws IOException, KeeperException, InterruptedException {
312       super(conf, csm);
313     }
314 
315     @Override
316     public void run() {
317       super.run();
318       if (this.zkcluster != null) {
319         try {
320           this.zkcluster.shutdown();
321         } catch (IOException e) {
322           e.printStackTrace();
323         }
324       }
325     }
326 
327     void setZKCluster(final MiniZooKeeperCluster zkcluster) {
328       this.zkcluster = zkcluster;
329     }
330   }
331 }