001/*
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hbase;
019
020import java.io.IOException;
021import java.util.Map;
022import org.apache.hadoop.conf.Configuration;
023import org.apache.hadoop.hbase.util.VersionInfo;
024import org.apache.hadoop.hbase.zookeeper.ZKConfig;
025import org.apache.yetus.audience.InterfaceAudience;
026import org.slf4j.Logger;
027import org.slf4j.LoggerFactory;
028
029/**
030 * Adds HBase configuration files to a Configuration
031 */
032@InterfaceAudience.Public
033public class HBaseConfiguration extends Configuration {
034  private static final Logger LOG = LoggerFactory.getLogger(HBaseConfiguration.class);
035
036  static {
037    addDeprecatedKeys();
038  }
039
040  /**
041   * Instantiating HBaseConfiguration() is deprecated. Please use HBaseConfiguration#create() to
042   * construct a plain Configuration
043   * @deprecated since 0.90.0. Please use {@link #create()} instead.
044   * @see #create()
045   * @see <a href="https://issues.apache.org/jira/browse/HBASE-2036">HBASE-2036</a>
046   */
047  @Deprecated
048  public HBaseConfiguration() {
049    // TODO:replace with private constructor, HBaseConfiguration should not extend Configuration
050    super();
051    addHbaseResources(this);
052    LOG.warn("instantiating HBaseConfiguration() is deprecated. Please use"
053      + " HBaseConfiguration#create() to construct a plain Configuration");
054  }
055
056  /**
057   * Instantiating HBaseConfiguration() is deprecated. Please use HBaseConfiguration#create(conf) to
058   * construct a plain Configuration
059   * @deprecated since 0.90.0. Please use {@link #create(Configuration)} instead.
060   * @see #create(Configuration)
061   * @see <a href="https://issues.apache.org/jira/browse/HBASE-2036">HBASE-2036</a>
062   */
063  @Deprecated
064  public HBaseConfiguration(final Configuration c) {
065    // TODO:replace with private constructor
066    this();
067    merge(this, c);
068  }
069
070  private static void checkDefaultsVersion(Configuration conf) {
071    if (conf.getBoolean("hbase.defaults.for.version.skip", Boolean.FALSE)) return;
072    String defaultsVersion = conf.get("hbase.defaults.for.version");
073    String thisVersion = VersionInfo.getVersion();
074    if (!thisVersion.equals(defaultsVersion)) {
075      throw new RuntimeException(
076        "hbase-default.xml file seems to be for an older version of HBase (" + defaultsVersion
077          + "), this version is " + thisVersion);
078    }
079  }
080
081  /**
082   * The hbase.ipc.server.reservoir.initial.max and hbase.ipc.server.reservoir.initial.buffer.size
083   * were introduced in HBase2.0.0, while in HBase3.0.0 the two config keys will be replaced by
084   * hbase.server.allocator.max.buffer.count and hbase.server.allocator.buffer.size. Also the
085   * hbase.ipc.server.reservoir.enabled will be replaced by hbase.server.allocator.pool.enabled.
086   * Keep the three old config keys here for HBase2.x compatibility. <br>
087   * HBASE-24667: This config hbase.regionserver.hostname.disable.master.reversedns will be replaced
088   * by hbase.unsafe.regionserver.hostname.disable.master.reversedns. Keep the old config keys here
089   * for backward compatibility. <br>
090   * Note: Before Hadoop-3.3, we must call the addDeprecations method before creating the
091   * Configuration object to work correctly. After this bug is fixed in hadoop-3.3, there will be no
092   * order problem.
093   * @see <a href="https://issues.apache.org/jira/browse/HADOOP-15708">HADOOP-15708</a>
094   */
095  private static void addDeprecatedKeys() {
096    Configuration.addDeprecations(new DeprecationDelta[] {
097      new DeprecationDelta("hbase.regionserver.hostname", "hbase.unsafe.regionserver.hostname"),
098      new DeprecationDelta("hbase.regionserver.hostname.disable.master.reversedns",
099        "hbase.unsafe.regionserver.hostname.disable.master.reversedns"),
100      new DeprecationDelta("hbase.offheapcache.minblocksize", "hbase.blockcache.minblocksize"),
101      new DeprecationDelta("hbase.ipc.server.reservoir.enabled",
102        "hbase.server.allocator.pool.enabled"),
103      new DeprecationDelta("hbase.ipc.server.reservoir.initial.max",
104        "hbase.server.allocator.max.buffer.count"),
105      new DeprecationDelta("hbase.ipc.server.reservoir.initial.buffer.size",
106        "hbase.server.allocator.buffer.size"),
107      new DeprecationDelta("hlog.bulk.output", "wal.bulk.output"),
108      new DeprecationDelta("hlog.input.tables", "wal.input.tables"),
109      new DeprecationDelta("hlog.input.tablesmap", "wal.input.tablesmap"), new DeprecationDelta(
110        "hbase.normalizer.min.region.count", "hbase.normalizer.merge.min.region.count") });
111  }
112
113  public static Configuration addHbaseResources(Configuration conf) {
114    conf.addResource("hbase-default.xml");
115    conf.addResource("hbase-site.xml");
116
117    checkDefaultsVersion(conf);
118    return conf;
119  }
120
121  /**
122   * Creates a Configuration with HBase resources
123   * @return a Configuration with HBase resources
124   */
125  public static Configuration create() {
126    Configuration conf = new Configuration();
127    // In case HBaseConfiguration is loaded from a different classloader than
128    // Configuration, conf needs to be set with appropriate class loader to resolve
129    // HBase resources.
130    conf.setClassLoader(HBaseConfiguration.class.getClassLoader());
131    return addHbaseResources(conf);
132  }
133
134  /**
135   * Creates a Configuration with HBase resources
136   * @param that Configuration to clone.
137   * @return a Configuration created with the hbase-*.xml files plus the given configuration.
138   */
139  public static Configuration create(final Configuration that) {
140    Configuration conf = create();
141    merge(conf, that);
142    return conf;
143  }
144
145  /**
146   * Merge two configurations.
147   * @param destConf the configuration that will be overwritten with items from the srcConf
148   * @param srcConf  the source configuration
149   **/
150  public static void merge(Configuration destConf, Configuration srcConf) {
151    for (Map.Entry<String, String> e : srcConf) {
152      destConf.set(e.getKey(), e.getValue());
153    }
154  }
155
156  /**
157   * Returns a subset of the configuration properties, matching the given key prefix. The prefix is
158   * stripped from the return keys, ie. when calling with a prefix of "myprefix", the entry
159   * "myprefix.key1 = value1" would be returned as "key1 = value1". If an entry's key matches the
160   * prefix exactly ("myprefix = value2"), it will <strong>not</strong> be included in the results,
161   * since it would show up as an entry with an empty key.
162   */
163  public static Configuration subset(Configuration srcConf, String prefix) {
164    Configuration newConf = new Configuration(false);
165    for (Map.Entry<String, String> entry : srcConf) {
166      if (entry.getKey().startsWith(prefix)) {
167        String newKey = entry.getKey().substring(prefix.length());
168        // avoid entries that would produce an empty key
169        if (!newKey.isEmpty()) {
170          newConf.set(newKey, entry.getValue());
171        }
172      }
173    }
174    return newConf;
175  }
176
177  /**
178   * Sets all the entries in the provided {@code Map<String, String>} as properties in the given
179   * {@code Configuration}. Each property will have the specified prefix prepended, so that the
180   * configuration entries are keyed by {@code prefix + entry.getKey()}.
181   */
182  public static void setWithPrefix(Configuration conf, String prefix,
183    Iterable<Map.Entry<String, String>> properties) {
184    for (Map.Entry<String, String> entry : properties) {
185      conf.set(prefix + entry.getKey(), entry.getValue());
186    }
187  }
188
189  /** Returns whether to show HBase Configuration in servlet */
190  public static boolean isShowConfInServlet() {
191    boolean isShowConf = false;
192    try {
193      if (Class.forName("org.apache.hadoop.conf.ConfServlet") != null) {
194        isShowConf = true;
195      }
196    } catch (LinkageError e) {
197      // should we handle it more aggressively in addition to log the error?
198      LOG.warn("Error thrown: ", e);
199    } catch (ClassNotFoundException ce) {
200      LOG.debug("ClassNotFound: ConfServlet");
201      // ignore
202    }
203    return isShowConf;
204  }
205
206  /**
207   * Get the value of the <code>name</code> property as an <code>int</code>, possibly referring to
208   * the deprecated name of the configuration property. If no such property exists, the provided
209   * default value is returned, or if the specified value is not a valid <code>int</code>, then an
210   * error is thrown.
211   * @param name           property name.
212   * @param deprecatedName a deprecatedName for the property to use if non-deprecated name is not
213   *                       used
214   * @param defaultValue   default value.
215   * @throws NumberFormatException when the value is invalid
216   * @return property value as an <code>int</code>, or <code>defaultValue</code>.
217   * @deprecated it will be removed in 3.0.0. Use
218   *             {@link Configuration#addDeprecation(String, String)} instead.
219   */
220  @Deprecated
221  public static int getInt(Configuration conf, String name, String deprecatedName,
222    int defaultValue) {
223    if (conf.get(deprecatedName) != null) {
224      LOG.warn(String.format("Config option \"%s\" is deprecated. Instead, use \"%s\"",
225        deprecatedName, name));
226      return conf.getInt(deprecatedName, defaultValue);
227    } else {
228      return conf.getInt(name, defaultValue);
229    }
230  }
231
232  /**
233   * Get the password from the Configuration instance using the getPassword method if it exists. If
234   * not, then fall back to the general get method for configuration elements.
235   * @param conf    configuration instance for accessing the passwords
236   * @param alias   the name of the password element
237   * @param defPass the default password
238   * @return String password or default password
239   */
240  public static String getPassword(Configuration conf, String alias, String defPass)
241    throws IOException {
242    String passwd;
243    char[] p = conf.getPassword(alias);
244    if (p != null) {
245      LOG.debug("Config option {} was found through the Configuration getPassword method.", alias);
246      passwd = new String(p);
247    } else {
248      LOG.debug("Config option {} was not found. Using provided default value", alias);
249      passwd = defPass;
250    }
251    return passwd;
252  }
253
254  /**
255   * Generates a {@link Configuration} instance by applying the ZooKeeper cluster key to the base
256   * Configuration. Note that additional configuration properties may be needed for a remote
257   * cluster, so it is preferable to use {@link #createClusterConf(Configuration, String, String)}.
258   * @param baseConf   the base configuration to use, containing prefixed override properties
259   * @param clusterKey the ZooKeeper quorum cluster key to apply, or {@code null} if none
260   * @return the merged configuration with override properties and cluster key applied
261   * @see #createClusterConf(Configuration, String, String)
262   */
263  public static Configuration createClusterConf(Configuration baseConf, String clusterKey)
264    throws IOException {
265    return createClusterConf(baseConf, clusterKey, null);
266  }
267
268  /**
269   * Generates a {@link Configuration} instance by applying property overrides prefixed by a cluster
270   * profile key to the base Configuration. Override properties are extracted by the
271   * {@link #subset(Configuration, String)} method, then the merged on top of the base Configuration
272   * and returned.
273   * @param baseConf       the base configuration to use, containing prefixed override properties
274   * @param clusterKey     the ZooKeeper quorum cluster key to apply, or {@code null} if none
275   * @param overridePrefix the property key prefix to match for override properties, or {@code null}
276   *                       if none
277   * @return the merged configuration with override properties and cluster key applied
278   */
279  public static Configuration createClusterConf(Configuration baseConf, String clusterKey,
280    String overridePrefix) throws IOException {
281    Configuration clusterConf = HBaseConfiguration.create(baseConf);
282    if (clusterKey != null && !clusterKey.isEmpty()) {
283      applyClusterKeyToConf(clusterConf, clusterKey);
284    }
285
286    if (overridePrefix != null && !overridePrefix.isEmpty()) {
287      Configuration clusterSubset = HBaseConfiguration.subset(clusterConf, overridePrefix);
288      HBaseConfiguration.merge(clusterConf, clusterSubset);
289    }
290    return clusterConf;
291  }
292
293  /**
294   * Apply the settings in the given key to the given configuration, this is used to communicate
295   * with distant clusters
296   * @param conf configuration object to configure
297   * @param key  string that contains the 3 required configuratins
298   */
299  private static void applyClusterKeyToConf(Configuration conf, String key) throws IOException {
300    ZKConfig.ZKClusterKey zkClusterKey = ZKConfig.transformClusterKey(key);
301    conf.set(HConstants.ZOOKEEPER_QUORUM, zkClusterKey.getQuorumString());
302    conf.setInt(HConstants.ZOOKEEPER_CLIENT_PORT, zkClusterKey.getClientPort());
303    conf.set(HConstants.ZOOKEEPER_ZNODE_PARENT, zkClusterKey.getZnodeParent());
304    // Without the right registry, the above configs are useless. Also, we don't use setClass()
305    // here because the ConnectionRegistry* classes are not resolvable from this module.
306    // This will be broken if ZkConnectionRegistry class gets renamed or moved. Is there a better
307    // way?
308    LOG.info("Overriding client registry implementation to {}",
309      HConstants.ZK_CONNECTION_REGISTRY_CLASS);
310    conf.set(HConstants.CLIENT_CONNECTION_REGISTRY_IMPL_CONF_KEY,
311      HConstants.ZK_CONNECTION_REGISTRY_CLASS);
312  }
313
314  /**
315   * For debugging. Dump configurations to system output as xml format. Master and RS configurations
316   * can also be dumped using http services. e.g. "curl http://master:16010/dump"
317   */
318  public static void main(String[] args) throws Exception {
319    HBaseConfiguration.create().writeXml(System.out);
320  }
321}