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.commons.lang3.StringUtils;
023import org.apache.hadoop.conf.Configuration;
024import org.apache.hadoop.hbase.util.VersionInfo;
025import org.apache.hadoop.hbase.zookeeper.ZKConfig;
026import org.apache.yetus.audience.InterfaceAudience;
027import org.slf4j.Logger;
028import org.slf4j.LoggerFactory;
029
030/**
031 * Adds HBase configuration files to a Configuration
032 */
033@InterfaceAudience.Public
034public class HBaseConfiguration extends Configuration {
035  private static final Logger LOG = LoggerFactory.getLogger(HBaseConfiguration.class);
036
037  static {
038    addDeprecatedKeys();
039  }
040
041  private static void checkDefaultsVersion(Configuration conf) {
042    if (conf.getBoolean("hbase.defaults.for.version.skip", Boolean.FALSE)) return;
043    String defaultsVersion = conf.get("hbase.defaults.for.version");
044    String thisVersion = VersionInfo.getVersion();
045    if (!thisVersion.equals(defaultsVersion)) {
046      throw new RuntimeException(
047        "hbase-default.xml file seems to be for an older version of HBase (" + defaultsVersion
048          + "), this version is " + thisVersion);
049    }
050  }
051
052  /**
053   * The hbase.ipc.server.reservoir.initial.max and hbase.ipc.server.reservoir.initial.buffer.size
054   * were introduced in HBase2.0.0, while in HBase3.0.0 the two config keys will be replaced by
055   * hbase.server.allocator.max.buffer.count and hbase.server.allocator.buffer.size. Also the
056   * hbase.ipc.server.reservoir.enabled will be replaced by hbase.server.allocator.pool.enabled.
057   * Keep the three old config keys here for HBase2.x compatibility. <br>
058   * HBASE-24667: This config hbase.regionserver.hostname.disable.master.reversedns will be replaced
059   * by hbase.unsafe.regionserver.hostname.disable.master.reversedns. Keep the old config keys here
060   * for backward compatibility. <br>
061   * Note: Before Hadoop-3.3, we must call the addDeprecations method before creating the
062   * Configuration object to work correctly. After this bug is fixed in hadoop-3.3, there will be no
063   * order problem.
064   * @see <a href="https://issues.apache.org/jira/browse/HADOOP-15708">HADOOP-15708</a>
065   */
066  private static void addDeprecatedKeys() {
067    Configuration.addDeprecations(new DeprecationDelta[] {
068      new DeprecationDelta("hbase.regionserver.hostname", "hbase.unsafe.regionserver.hostname"),
069      new DeprecationDelta("hbase.regionserver.hostname.disable.master.reversedns",
070        "hbase.unsafe.regionserver.hostname.disable.master.reversedns"),
071      new DeprecationDelta("hbase.offheapcache.minblocksize", "hbase.blockcache.minblocksize"),
072      new DeprecationDelta("hbase.ipc.server.reservoir.enabled",
073        "hbase.server.allocator.pool.enabled"),
074      new DeprecationDelta("hbase.ipc.server.reservoir.initial.max",
075        "hbase.server.allocator.max.buffer.count"),
076      new DeprecationDelta("hbase.ipc.server.reservoir.initial.buffer.size",
077        "hbase.server.allocator.buffer.size"),
078      new DeprecationDelta("hlog.bulk.output", "wal.bulk.output"),
079      new DeprecationDelta("hlog.input.tables", "wal.input.tables"),
080      new DeprecationDelta("hlog.input.tablesmap", "wal.input.tablesmap"),
081      new DeprecationDelta("hbase.master.mob.ttl.cleaner.period",
082        "hbase.master.mob.cleaner.period"),
083      new DeprecationDelta("hbase.normalizer.min.region.count",
084        "hbase.normalizer.merge.min.region.count") });
085  }
086
087  public static Configuration addHbaseResources(Configuration conf) {
088    conf.addResource("hbase-default.xml");
089    conf.addResource("hbase-site.xml");
090
091    checkDefaultsVersion(conf);
092    return conf;
093  }
094
095  /**
096   * Creates a Configuration with HBase resources
097   * @return a Configuration with HBase resources
098   */
099  public static Configuration create() {
100    Configuration conf = new Configuration();
101    // In case HBaseConfiguration is loaded from a different classloader than
102    // Configuration, conf needs to be set with appropriate class loader to resolve
103    // HBase resources.
104    conf.setClassLoader(HBaseConfiguration.class.getClassLoader());
105    return addHbaseResources(conf);
106  }
107
108  /**
109   * Creates a Configuration with HBase resources
110   * @param that Configuration to clone.
111   * @return a Configuration created with the hbase-*.xml files plus the given configuration.
112   */
113  public static Configuration create(final Configuration that) {
114    Configuration conf = create();
115    merge(conf, that);
116    return conf;
117  }
118
119  /**
120   * Merge two configurations.
121   * @param destConf the configuration that will be overwritten with items from the srcConf
122   * @param srcConf  the source configuration
123   **/
124  public static void merge(Configuration destConf, Configuration srcConf) {
125    for (Map.Entry<String, String> e : srcConf) {
126      destConf.set(e.getKey(), e.getValue());
127    }
128  }
129
130  /**
131   * Returns a subset of the configuration properties, matching the given key prefix. The prefix is
132   * stripped from the return keys, ie. when calling with a prefix of "myprefix", the entry
133   * "myprefix.key1 = value1" would be returned as "key1 = value1". If an entry's key matches the
134   * prefix exactly ("myprefix = value2"), it will <strong>not</strong> be included in the results,
135   * since it would show up as an entry with an empty key.
136   */
137  public static Configuration subset(Configuration srcConf, String prefix) {
138    Configuration newConf = new Configuration(false);
139    for (Map.Entry<String, String> entry : srcConf) {
140      if (entry.getKey().startsWith(prefix)) {
141        String newKey = entry.getKey().substring(prefix.length());
142        // avoid entries that would produce an empty key
143        if (!newKey.isEmpty()) {
144          newConf.set(newKey, entry.getValue());
145        }
146      }
147    }
148    return newConf;
149  }
150
151  /**
152   * Sets all the entries in the provided {@code Map<String, String>} as properties in the given
153   * {@code Configuration}. Each property will have the specified prefix prepended, so that the
154   * configuration entries are keyed by {@code prefix + entry.getKey()}.
155   */
156  public static void setWithPrefix(Configuration conf, String prefix,
157    Iterable<Map.Entry<String, String>> properties) {
158    for (Map.Entry<String, String> entry : properties) {
159      conf.set(prefix + entry.getKey(), entry.getValue());
160    }
161  }
162
163  /** Returns whether to show HBase Configuration in servlet */
164  public static boolean isShowConfInServlet() {
165    boolean isShowConf = false;
166    try {
167      if (Class.forName("org.apache.hadoop.conf.ConfServlet") != null) {
168        isShowConf = true;
169      }
170    } catch (LinkageError e) {
171      // should we handle it more aggressively in addition to log the error?
172      LOG.warn("Error thrown: ", e);
173    } catch (ClassNotFoundException ce) {
174      LOG.debug("ClassNotFound: ConfServlet");
175      // ignore
176    }
177    return isShowConf;
178  }
179
180  /**
181   * Get the password from the Configuration instance using the getPassword method if it exists. If
182   * not, then fall back to the general get method for configuration elements.
183   * @param conf    configuration instance for accessing the passwords
184   * @param alias   the name of the password element
185   * @param defPass the default password
186   * @return String password or default password
187   */
188  public static String getPassword(Configuration conf, String alias, String defPass)
189    throws IOException {
190    String passwd;
191    char[] p = conf.getPassword(alias);
192    if (p != null) {
193      LOG.debug("Config option {} was found through the Configuration getPassword method.", alias);
194      passwd = new String(p);
195    } else {
196      LOG.debug("Config option {} was not found. Using provided default value", alias);
197      passwd = defPass;
198    }
199    return passwd;
200  }
201
202  /**
203   * Generates a {@link Configuration} instance by applying the ZooKeeper cluster key to the base
204   * Configuration. Note that additional configuration properties may be needed for a remote
205   * cluster, so it is preferable to use {@link #createClusterConf(Configuration, String, String)}.
206   * @param baseConf   the base configuration to use, containing prefixed override properties
207   * @param clusterKey the ZooKeeper quorum cluster key to apply, or {@code null} if none
208   * @return the merged configuration with override properties and cluster key applied
209   * @see #createClusterConf(Configuration, String, String)
210   */
211  public static Configuration createClusterConf(Configuration baseConf, String clusterKey)
212    throws IOException {
213    return createClusterConf(baseConf, clusterKey, null);
214  }
215
216  /**
217   * Generates a {@link Configuration} instance by applying property overrides prefixed by a cluster
218   * profile key to the base Configuration. Override properties are extracted by the
219   * {@link #subset(Configuration, String)} method, then the merged on top of the base Configuration
220   * and returned.
221   * @param baseConf       the base configuration to use, containing prefixed override properties
222   * @param clusterKey     the ZooKeeper quorum cluster key to apply, or {@code null} if none
223   * @param overridePrefix the property key prefix to match for override properties, or {@code null}
224   *                       if none
225   * @return the merged configuration with override properties and cluster key applied
226   */
227  public static Configuration createClusterConf(Configuration baseConf, String clusterKey,
228    String overridePrefix) throws IOException {
229    Configuration clusterConf = HBaseConfiguration.create(baseConf);
230    if (!StringUtils.isBlank(clusterKey)) {
231      applyClusterKeyToConf(clusterConf, clusterKey);
232    }
233
234    if (!StringUtils.isBlank(overridePrefix)) {
235      Configuration clusterSubset = HBaseConfiguration.subset(clusterConf, overridePrefix);
236      HBaseConfiguration.merge(clusterConf, clusterSubset);
237    }
238    return clusterConf;
239  }
240
241  /**
242   * Apply the settings in the given key to the given configuration, this is used to communicate
243   * with distant clusters
244   * @param conf configuration object to configure
245   * @param key  string that contains the 3 required configuratins
246   */
247  private static void applyClusterKeyToConf(Configuration conf, String key) throws IOException {
248    ZKConfig.ZKClusterKey zkClusterKey = ZKConfig.transformClusterKey(key);
249    conf.set(HConstants.ZOOKEEPER_QUORUM, zkClusterKey.getQuorumString());
250    conf.setInt(HConstants.ZOOKEEPER_CLIENT_PORT, zkClusterKey.getClientPort());
251    conf.set(HConstants.ZOOKEEPER_ZNODE_PARENT, zkClusterKey.getZnodeParent());
252    // Without the right registry, the above configs are useless. Also, we don't use setClass()
253    // here because the ConnectionRegistry* classes are not resolvable from this module.
254    // This will be broken if ZkConnectionRegistry class gets renamed or moved. Is there a better
255    // way?
256    LOG.info("Overriding client registry implementation to {}",
257      HConstants.ZK_CONNECTION_REGISTRY_CLASS);
258    conf.set(HConstants.CLIENT_CONNECTION_REGISTRY_IMPL_CONF_KEY,
259      HConstants.ZK_CONNECTION_REGISTRY_CLASS);
260  }
261
262  /**
263   * For debugging. Dump configurations to system output as xml format. Master and RS configurations
264   * can also be dumped using http services. e.g. "curl http://master:16010/dump"
265   */
266  public static void main(String[] args) throws Exception {
267    HBaseConfiguration.create().writeXml(System.out);
268  }
269}