diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/HColumnDescriptor.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/HColumnDescriptor.java index ecb0826..d012863 100644 --- a/hbase-client/src/main/java/org/apache/hadoop/hbase/HColumnDescriptor.java +++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/HColumnDescriptor.java @@ -118,7 +118,7 @@ public class HColumnDescriptor implements WritableComparable /** * Default number of versions of a record to keep. */ - public static final int DEFAULT_VERSIONS = 3; + public static final int DEFAULT_VERSIONS = 1; /** * Default is not to keep a minimum of versions. @@ -151,7 +151,7 @@ public class HColumnDescriptor implements WritableComparable * is enabled. */ public static final boolean DEFAULT_CACHE_DATA_ON_WRITE = false; - + /** * Default setting for whether to cache index blocks on write if block * caching is enabled. @@ -166,7 +166,7 @@ public class HColumnDescriptor implements WritableComparable /** * Default setting for whether or not to use bloomfilters. */ - public static final String DEFAULT_BLOOMFILTER = BloomType.NONE.toString(); + public static final String DEFAULT_BLOOMFILTER = BloomType.ROW.toString(); /** * Default setting for whether to cache bloom filter blocks on write if block @@ -543,7 +543,7 @@ public class HColumnDescriptor implements WritableComparable return Compression.Algorithm.valueOf(n.toUpperCase()); } - /** @return compression type being used for the column family for major + /** @return compression type being used for the column family for major compression */ public Compression.Algorithm getCompactionCompression() { String n = getValue(COMPRESSION_COMPACT); diff --git a/hbase-client/src/test/java/org/apache/hadoop/hbase/client/TestClientNoCluster.java b/hbase-client/src/test/java/org/apache/hadoop/hbase/client/TestClientNoCluster.java index 8a541e2..fec02be 100644 --- a/hbase-client/src/test/java/org/apache/hadoop/hbase/client/TestClientNoCluster.java +++ b/hbase-client/src/test/java/org/apache/hadoop/hbase/client/TestClientNoCluster.java @@ -87,6 +87,15 @@ public class TestClientNoCluster { } } + // Disabled. Used looking at timeouts and retries. @Test + public void testTimeout() throws IOException { + this.conf.set("hbase.client.connection.impl", TimeoutConnection.class.getName()); + Configuration localConfig = HBaseConfiguration.create(this.conf); + localConfig.setInt("hbase.client.pause", 100); + localConfig.setInt("hbase.client.retries.number", 20); + MetaScanner.metaScan(localConfig, null); + } + @Test public void testDoNotRetryMetaScanner() throws IOException { this.conf.set("hbase.client.connection.impl", @@ -197,4 +206,34 @@ public class TestClientNoCluster { return this.stub; } } + + /** + * Override to just timeout a connection + */ + static class TimeoutConnection + extends HConnectionManager.HConnectionImplementation { + final ClientService.BlockingInterface stub; + + TimeoutConnection(Configuration conf, boolean managed) + throws IOException { + super(conf, managed); + // Mock up my stub so open scanner returns a scanner id and then on next, we throw + // exceptions for three times and then after that, we return no more to scan. + this.stub = Mockito.mock(ClientService.BlockingInterface.class); + long sid = 12345L; + try { + Mockito.when(stub.scan((RpcController)Mockito.any(), + (ClientProtos.ScanRequest)Mockito.any())). + thenReturn(ClientProtos.ScanResponse.newBuilder().setScannerId(sid).build()). + thenThrow(new ServiceException(new RegionServerStoppedException("From Mockito"))); + } catch (ServiceException e) { + throw new IOException(e); + } + } + + @Override + public BlockingInterface getClient(ServerName sn) throws IOException { + return this.stub; + } + } } \ No newline at end of file diff --git a/hbase-common/src/main/resources/hbase-default.xml b/hbase-common/src/main/resources/hbase-default.xml index f0a19e8..b47231f 100644 --- a/hbase-common/src/main/resources/hbase-default.xml +++ b/hbase-common/src/main/resources/hbase-default.xml @@ -19,26 +19,52 @@ * limitations under the License. */ --> + + + + + + hbase.tmp.dir + ${java.io.tmpdir}/hbase-${user.name} + Temporary directory on the local filesystem. + Change this setting to point to a location more permanent + than '/tmp', the usual resolve for java.io.tmpdir, as the + '/tmp' directory is cleared on machine restart. + + hbase.rootdir - file:///tmp/hbase-${user.name}/hbase + file://${hbase.tmp.dir}/hbase The directory shared by region servers and into which HBase persists. The URL should be 'fully-qualified' to include the filesystem scheme. For example, to specify the HDFS directory '/hbase' where the HDFS instance's namenode is running at namenode.example.org on port 9000, set this value to: - hdfs://namenode.example.org:9000/hbase. By default HBase writes - into /tmp. Change this configuration else all data will be lost - on machine restart. + hdfs://namenode.example.org:9000/hbase. By default, we write + to whatever ${hbase.tmp.dir} is set too -- usually /tmp -- + so change this configuration or else all data will be lost on + machine restart. - hbase.master.port - 60000 - The port the HBase Master should bind to. - - hbase.cluster.distributed false The mode the cluster will be in. Possible values are @@ -48,14 +74,20 @@ - hbase.tmp.dir - ${java.io.tmpdir}/hbase-${user.name} - Temporary directory on the local filesystem. - Change this setting to point to a location more permanent - than '/tmp' (The '/tmp' directory is often cleared on - machine restart). + hbase.zookeeper.quorum + localhost + Comma separated list of servers in the ZooKeeper Quorum. + For example, "host1.mydomain.com,host2.mydomain.com,host3.mydomain.com". + By default this is set to localhost for local and pseudo-distributed modes + of operation. For a fully-distributed setup, this should be set to a full + list of ZooKeeper quorum servers. If HBASE_MANAGES_ZK is set in hbase-env.sh + this is the list of servers which hbase will start/stop ZooKeeper on as + part of cluster start/stop. + + hbase.local.dir ${hbase.tmp.dir}/local/ @@ -63,6 +95,13 @@ as a local storage. + + + + hbase.master.port + 60000 + The port the HBase Master should bind to. + hbase.master.info.port 60010 @@ -77,101 +116,90 @@ - hbase.client.write.buffer - 2097152 - Default size of the HTable clien write buffer in bytes. - A bigger buffer takes more memory -- on both the client and server - side since server instantiates the passed write buffer to process - it -- but a larger buffer size reduces the number of RPCs made. - For an estimate of server-side memory-used, evaluate - hbase.client.write.buffer * hbase.regionserver.handler.count + hbase.master.logcleaner.plugins + org.apache.hadoop.hbase.master.cleaner.TimeToLiveLogCleaner + A comma-separated list of LogCleanerDelegate invoked by + the LogsCleaner service. These WAL/HLog cleaners are called in order, + so put the HLog cleaner that prunes the most HLog files in front. To + implement your own LogCleanerDelegate, just put it in HBase's classpath + and add the fully qualified class name here. Always add the above + default log cleaners in the list. - hbase.regionserver.port - 60020 - The port the HBase RegionServer binds to. + hbase.master.logcleaner.ttl + 600000 + Maximum time a HLog can stay in the .oldlogdir directory, + after which it will be cleaned by a Master thread. - hbase.regionserver.info.port - 60030 - The port for the HBase RegionServer web UI - Set to -1 if you do not want the RegionServer UI to run. + hbase.master.hfilecleaner.plugins + org.apache.hadoop.hbase.master.cleaner.TimeToLiveHFileCleaner + A comma-separated list of HFileCleanerDelegate invoked by + the HFileCleaner service. These HFiles cleaners are called in order, + so put the cleaner that prunes the most files in front. To + implement your own HFileCleanerDelegate, just put it in HBase's classpath + and add the fully qualified class name here. Always add the above + default log cleaners in the list as they will be overwritten in hbase-site.xml. - hbase.regionserver.info.port.auto - false - Whether or not the Master or RegionServer - UI should search for a port to bind to. Enables automatic port - search if hbase.regionserver.info.port is already in use. - Useful for testing, turned off by default. - + hbase.master.catalog.timeout + 600000 + Timeout value for the Catalog Janitor from the master to META. - hbase.regionserver.info.bindAddress - 0.0.0.0 - The address for the HBase RegionServer web UI + hbase.master.dns.interface + default + The name of the Network Interface from which a master + should report its IP address. - hbase.client.pause - 1000 - General client pause value. Used mostly as value to wait - before running a retry of a failed get, region lookup, etc. - - - hbase.client.retries.number - 10 - Maximum retries. Used as maximum for all retryable - operations such as fetching of the root region from root region - server, getting a cell's value, starting a row update, etc. - Default: 10. + hbase.master.dns.nameserver + default + The host name or IP address of the name server (DNS) + which a master should use to determine the host name used + for communication and display purposes. + + - hbase.bulkload.retries.number - 0 - Maximum retries. This is maximum number of iterations - to atomic bulk loads are attempted in the face of splitting operations - 0 means never give up. Default: 0. + hbase.regionserver.port + 60020 + The port the HBase RegionServer binds to. - hbase.client.scanner.caching - 100 - Number of rows that will be fetched when calling next - on a scanner if it is not served from (local, client) memory. Higher - caching values will enable faster scanners but will eat up more memory - and some calls of next may take longer and longer times when the cache is empty. - Do not set this value such that the time between invocations is greater - than the scanner timeout; i.e. hbase.client.scanner.timeout.period + hbase.regionserver.info.port + 60030 + The port for the HBase RegionServer web UI + Set to -1 if you do not want the RegionServer UI to run. - hbase.client.keyvalue.maxsize - 10485760 - Specifies the combined maximum allowed size of a KeyValue - instance. This is to set an upper boundary for a single entry saved in a - storage file. Since they cannot be split it helps avoiding that a region - cannot be split any further because the data is too large. It seems wise - to set this to a fraction of the maximum region size. Setting it to zero - or less disables the check. + hbase.regionserver.info.bindAddress + 0.0.0.0 + The address for the HBase RegionServer web UI - hbase.client.scanner.timeout.period - 60000 - Client scanner lease period in milliseconds. Default is - 60 seconds. + hbase.regionserver.info.port.auto + false + Whether or not the Master or RegionServer + UI should search for a port to bind to. Enables automatic port + search if hbase.regionserver.info.port is already in use. + Useful for testing, turned off by default. + hbase.regionserver.handler.count - 10 + 30 Count of RPC Listener instances spun up on RegionServers. Same property is used by the Master for count of master handlers. - Default is 10. + Default is 30. @@ -224,28 +252,38 @@ The HLog file writer implementation. - hbase.regionserver.nbreservationblocks - 4 - The number of resevoir blocks of memory release on - OOME so we can cleanup properly before server shutdown. + hbase.regionserver.global.memstore.upperLimit + 0.4 + Maximum size of all memstores in a region server before new + updates are blocked and flushes are forced. Defaults to 40% of heap. + Updates are blocked and flushes are forced until size of all memstores + in a region server hits hbase.regionserver.global.memstore.lowerLimit. - hbase.zookeeper.dns.interface - default - The name of the Network Interface from which a ZooKeeper server - should report its IP address. + hbase.regionserver.global.memstore.lowerLimit + 0.38 + Maximum size of all memstores in a region server before + flushes are forced. Defaults to 38% of heap. + This value equal to hbase.regionserver.global.memstore.upperLimit causes + the minimum possible flushing to occur when updates are blocked due to + memstore limiting. - hbase.zookeeper.dns.nameserver - default - The host name or IP address of the name server (DNS) - which a ZooKeeper server should use to determine the host name used by the - master for communication and display purposes. + hbase.regionserver.optionalcacheflushinterval + 3600000 + + Maximum amount of time an edit lives in memory before being automatically flushed. + Default 1 hour. Set it to 0 to disable automatic flushing. + hbase.regionserver.catalog.timeout + 600000 + Timeout value for the Catalog Janitor from the regionserver to META. + + hbase.regionserver.dns.interface default The name of the Network Interface from which a region server @@ -260,70 +298,226 @@ master for communication and display purposes. + + - hbase.master.dns.interface + zookeeper.session.timeout + 90000 + ZooKeeper session timeout. + HBase passes this to the zk quorum as suggested maximum time for a + session (This setting becomes zookeeper's 'maxSessionTimeout'). See + http://hadoop.apache.org/zookeeper/docs/current/zookeeperProgrammers.html#ch_zkSessions + "The client sends a requested timeout, the server responds with the + timeout that it can give the client. " In milliseconds. + + + + zookeeper.znode.parent + /hbase + Root ZNode for HBase in ZooKeeper. All of HBase's ZooKeeper + files that are configured with a relative path will go under this node. + By default, all of HBase's ZooKeeper file path are configured with a + relative path, so they will all go under this directory unless changed. + + + + zookeeper.znode.rootserver + root-region-server + Path to ZNode holding root region location. This is written by + the master and read by clients and region servers. If a relative path is + given, the parent folder will be ${zookeeper.znode.parent}. By default, + this means the root location is stored at /hbase/root-region-server. + + + + zookeeper.znode.acl.parent + acl + Root ZNode for access control lists. + + + hbase.zookeeper.dns.interface default - The name of the Network Interface from which a master + The name of the Network Interface from which a ZooKeeper server should report its IP address. - hbase.master.dns.nameserver + hbase.zookeeper.dns.nameserver default The host name or IP address of the name server (DNS) - which a master should use to determine the host name used - for communication and display purposes. + which a ZooKeeper server should use to determine the host name used by the + master for communication and display purposes. + - hbase.balancer.period - - 300000 - Period at which the region balancer runs in the Master. + hbase.zookeeper.peerport + 2888 + Port used by ZooKeeper peers to talk to each other. + See http://hadoop.apache.org/zookeeper/docs/r3.1.1/zookeeperStarted.html#sc_RunningReplicatedZooKeeper + for more information. - hbase.regions.slop - 0.2 - Rebalance if any regionserver has average + (average * slop) regions. - Default is 20% slop. + hbase.zookeeper.leaderport + 3888 + Port used by ZooKeeper for leader election. + See http://hadoop.apache.org/zookeeper/docs/r3.1.1/zookeeperStarted.html#sc_RunningReplicatedZooKeeper + for more information. + - hbase.master.logcleaner.ttl - 600000 - Maximum time a HLog can stay in the .oldlogdir directory, - after which it will be cleaned by a Master thread. + hbase.zookeeper.useMulti + false + Instructs HBase to make use of ZooKeeper's multi-update functionality. + This allows certain ZooKeeper operations to complete more quickly and prevents some issues + with rare Replication failure scenarios (see the release note of HBASE-2611 for an example). + IMPORTANT: only set this to true if all ZooKeeper servers in the cluster are on version 3.4+ + and will not be downgraded. ZooKeeper versions before 3.4 do not support multi-update and will + not fail gracefully if multi-update is invoked (see ZOOKEEPER-1495). - hbase.master.logcleaner.plugins - org.apache.hadoop.hbase.master.cleaner.TimeToLiveLogCleaner - A comma-separated list of LogCleanerDelegate invoked by - the LogsCleaner service. These WAL/HLog cleaners are called in order, - so put the HLog cleaner that prunes the most HLog files in front. To - implement your own LogCleanerDelegate, just put it in HBase's classpath - and add the fully qualified class name here. Always add the above - default log cleaners in the list. + hbase.config.read.zookeeper.config + false + + Set to true to allow HBaseConfiguration to read the + zoo.cfg file for ZooKeeper properties. Switching this to true + is not recommended, since the functionality of reading ZK + properties from a zoo.cfg file has been deprecated. + + + + + hbase.zookeeper.property.initLimit + 10 + Property from ZooKeeper's config zoo.cfg. + The number of ticks that the initial synchronization phase can take. - hbase.regionserver.global.memstore.upperLimit - 0.4 - Maximum size of all memstores in a region server before new - updates are blocked and flushes are forced. Defaults to 40% of heap. - Updates are blocked and flushes are forced until size of all memstores - in a region server hits hbase.regionserver.global.memstore.lowerLimit. + hbase.zookeeper.property.syncLimit + 5 + Property from ZooKeeper's config zoo.cfg. + The number of ticks that can pass between sending a request and getting an + acknowledgment. - hbase.regionserver.global.memstore.lowerLimit - 0.35 - Maximum size of all memstores in a region server before - flushes are forced. Defaults to 35% of heap. - This value equal to hbase.regionserver.global.memstore.upperLimit causes - the minimum possible flushing to occur when updates are blocked due to - memstore limiting. + hbase.zookeeper.property.dataDir + ${hbase.tmp.dir}/zookeeper + Property from ZooKeeper's config zoo.cfg. + The directory where the snapshot is stored. + + + + hbase.zookeeper.property.clientPort + 2181 + Property from ZooKeeper's config zoo.cfg. + The port at which the clients will connect. + + + + hbase.zookeeper.property.maxClientCnxns + 300 + Property from ZooKeeper's config zoo.cfg. + Limit on number of concurrent connections (at the socket level) that a + single client, identified by IP address, may make to a single member of + the ZooKeeper ensemble. Set high to avoid zk connection issues running + standalone and pseudo-distributed. + + + + + + + hbase.client.write.buffer + 2097152 + Default size of the HTable client write buffer in bytes. + A bigger buffer takes more memory -- on both the client and server + side since server instantiates the passed write buffer to process + it -- but a larger buffer size reduces the number of RPCs made. + For an estimate of server-side memory-used, evaluate + hbase.client.write.buffer * hbase.regionserver.handler.count + + + + hbase.client.pause + 100 + General client pause value. Used mostly as value to wait + before running a retry of a failed get, region lookup, etc. + + + hbase.client.retries.number + 20 + Maximum retries. Used as maximum for all retryable + operations such as the getting of a cell's value, starting a row update, etc. + Default: 10. + + + + hbase.client.scanner.caching + 100 + Number of rows that will be fetched when calling next + on a scanner if it is not served from (local, client) memory. Higher + caching values will enable faster scanners but will eat up more memory + and some calls of next may take longer and longer times when the cache is empty. + Do not set this value such that the time between invocations is greater + than the scanner timeout; i.e. hbase.client.scanner.timeout.period + + + + hbase.client.keyvalue.maxsize + 10485760 + Specifies the combined maximum allowed size of a KeyValue + instance. This is to set an upper boundary for a single entry saved in a + storage file. Since they cannot be split it helps avoiding that a region + cannot be split any further because the data is too large. It seems wise + to set this to a fraction of the maximum region size. Setting it to zero + or less disables the check. + + + + hbase.client.scanner.timeout.period + 60000 + Client scanner lease period in milliseconds. Default is + 60 seconds. + + + + + hbase.bulkload.retries.number + 0 + Maximum retries. This is maximum number of iterations + to atomic bulk loads are attempted in the face of splitting operations + 0 means never give up. Default: 0. + + + + hbase.balancer.period + + 300000 + Period at which the region balancer runs in the Master. + + + + hbase.regions.slop + 0.2 + Rebalance if any regionserver has average + (average * slop) regions. + Default is 20% slop. @@ -343,14 +537,6 @@ - hbase.regionserver.optionalcacheflushinterval - 3600000 - - Maximum amount of time an edit lives in memory before being automatically flushed. - Default 1 hour. Set it to 0 to disable automatic flushing. - - - hbase.hregion.memstore.flush.size 134217728 @@ -406,6 +592,17 @@ + hbase.hregion.majorcompaction + 0 + The time (in miliseconds) between 'major' compactions of all + HStoreFiles in a region. Default: Set to 0 so we do not run major compactions. + Major compactions tend to happen exactly when you need them least so enable + them such that they run at off-peak for your deploy; or, since this setting is + on a periodicity that is unlikely to match your loading, run the compactions + via an external invocation out of a cron job or some such. + + + hbase.hstore.compactionThreshold 3 @@ -417,7 +614,7 @@ hbase.hstore.blockingStoreFiles - 7 + 10 If more than this number of StoreFiles in any one Store (one StoreFile is written per flush of MemStore) then updates are @@ -442,14 +639,6 @@ - hbase.hregion.majorcompaction - 86400000 - The time (in miliseconds) between 'major' compactions of all - HStoreFiles in a region. Default: 1 day. - Set to 0 to disable automated major compactions. - - - hbase.storescanner.parallel.seek.enable false @@ -479,19 +668,12 @@ hfile.block.cache.size - 0.25 + 0.4 Percentage of maximum heap (-Xmx setting) to allocate to block cache - used by HFile/StoreFile. Default of 0.25 means allocate 25%. - Set to 0 to disable but it's not recommended. - - - - hbase.hash.type - murmur - The hashing algorithm for use in HashFunction. Two values are - supported now: murmur (MurmurHash) and jenkins (JenkinsHash). - Used by bloom filters. + used by HFile/StoreFile. Default of 0.4 means allocate 40%. + Set to 0 to disable but it's not recommended; you need at least + enough cache to hold the storefile indices. @@ -521,20 +703,20 @@ + hfile.block.bloom.cacheonwrite + false + + Enables cache-on-write for inline blocks of a compound Bloom filter. + + + io.storefile.bloom.block.size 131072 The size in bytes of a single block ("chunk") of a compound Bloom filter. This size is approximate, because Bloom blocks can only be inserted at data block boundaries, and the number of keys per data - block varies. - - - - hfile.block.bloom.cacheonwrite - false - - Enables cache-on-write for inline blocks of a compound Bloom filter. + block varies. @@ -545,7 +727,6 @@ block is finished. - hbase.rpc.server.engine org.apache.hadoop.hbase.ipc.ProtobufRpcServerEngine @@ -554,13 +735,22 @@ + hbase.rpc.timeout + 10000 + + This is for the RPC layer to define how long HBase client applications + take for a remote call to time out. It uses pings to check connections + but will eventually throw a TimeoutException. + The default value is 10000ms(10s). + + + hbase.ipc.client.tcpnodelay true Set no delay on rpc socket connections. See http://docs.oracle.com/javase/1.5.0/docs/api/java/net/Socket.html#getTcpNoDelay() - @@ -599,7 +789,6 @@ specified in hbase.regionserver.keytab.file - hadoop.policy.file @@ -631,43 +820,6 @@ authentication token expires. Only used when HBase security is enabled. - - - zookeeper.session.timeout - 180000 - ZooKeeper session timeout. - HBase passes this to the zk quorum as suggested maximum time for a - session (This setting becomes zookeeper's 'maxSessionTimeout'). See - http://hadoop.apache.org/zookeeper/docs/current/zookeeperProgrammers.html#ch_zkSessions - "The client sends a requested timeout, the server responds with the - timeout that it can give the client. " In milliseconds. - - - - zookeeper.znode.parent - /hbase - Root ZNode for HBase in ZooKeeper. All of HBase's ZooKeeper - files that are configured with a relative path will go under this node. - By default, all of HBase's ZooKeeper file path are configured with a - relative path, so they will all go under this directory unless changed. - - - - zookeeper.znode.rootserver - root-region-server - Path to ZNode holding root region location. This is written by - the master and read by clients and region servers. If a relative path is - given, the parent folder will be ${zookeeper.znode.parent}. By default, - this means the root location is stored at /hbase/root-region-server. - - - - - zookeeper.znode.acl.parent - acl - Root ZNode for access control lists. - - hbase.coprocessor.region.classes @@ -678,114 +830,6 @@ A coprocessor can also be loaded on demand by setting HTableDescriptor. - - - hbase.coprocessor.master.classes - - A comma-separated list of - org.apache.hadoop.hbase.coprocessor.MasterObserver coprocessors that are - loaded by default on the active HMaster process. For any implemented - coprocessor methods, the listed classes will be called in order. After - implementing your own MasterObserver, just put it in HBase's classpath - and add the fully qualified class name here. - - - - - - hbase.zookeeper.quorum - localhost - Comma separated list of servers in the ZooKeeper Quorum. - For example, "host1.mydomain.com,host2.mydomain.com,host3.mydomain.com". - By default this is set to localhost for local and pseudo-distributed modes - of operation. For a fully-distributed setup, this should be set to a full - list of ZooKeeper quorum servers. If HBASE_MANAGES_ZK is set in hbase-env.sh - this is the list of servers which we will start/stop ZooKeeper on. - - - - hbase.zookeeper.peerport - 2888 - Port used by ZooKeeper peers to talk to each other. - See http://hadoop.apache.org/zookeeper/docs/r3.1.1/zookeeperStarted.html#sc_RunningReplicatedZooKeeper - for more information. - - - - hbase.zookeeper.leaderport - 3888 - Port used by ZooKeeper for leader election. - See http://hadoop.apache.org/zookeeper/docs/r3.1.1/zookeeperStarted.html#sc_RunningReplicatedZooKeeper - for more information. - - - - hbase.zookeeper.useMulti - false - Instructs HBase to make use of ZooKeeper's multi-update functionality. - This allows certain ZooKeeper operations to complete more quickly and prevents some issues - with rare Replication failure scenarios (see the release note of HBASE-2611 for an example). - IMPORTANT: only set this to true if all ZooKeeper servers in the cluster are on version 3.4+ - and will not be downgraded. ZooKeeper versions before 3.4 do not support multi-update and will - not fail gracefully if multi-update is invoked (see ZOOKEEPER-1495). - - - - - - - - hbase.zookeeper.property.initLimit - 10 - Property from ZooKeeper's config zoo.cfg. - The number of ticks that the initial synchronization phase can take. - - - - hbase.zookeeper.property.syncLimit - 5 - Property from ZooKeeper's config zoo.cfg. - The number of ticks that can pass between sending a request and getting an - acknowledgment. - - - - hbase.zookeeper.property.dataDir - ${hbase.tmp.dir}/zookeeper - Property from ZooKeeper's config zoo.cfg. - The directory where the snapshot is stored. - - - - hbase.zookeeper.property.clientPort - 2181 - Property from ZooKeeper's config zoo.cfg. - The port at which the clients will connect. - - - - hbase.zookeeper.property.maxClientCnxns - 300 - Property from ZooKeeper's config zoo.cfg. - Limit on number of concurrent connections (at the socket level) that a - single client, identified by IP address, may make to a single member of - the ZooKeeper ensemble. Set high to avoid zk connection issues running - standalone and pseudo-distributed. - - - hbase.rest.port 8080 @@ -800,7 +844,29 @@ true: Only the GET method is permitted. - + + hbase.rest.threads.max + 100 + + The maximum number of threads of the REST server thread pool. + Threads in the pool are reused to process REST requests. This + controls the maximum number of requests processed concurrently. + It may help to control the memory used by the REST server to + avoid OOM issues. If the thread pool is full, incoming requests + will be queued up and wait for some free threads. The default + is 100. + + + + hbase.rest.threads.min + 2 + + The minimum number of threads of the REST server thread pool. + The thread pool always has at least these number of threads so + the REST server is ready to serve incoming requests. The default + is 2. + + hbase.defaults.for.version @@@VERSION@@@ @@ -824,6 +890,17 @@ + hbase.coprocessor.master.classes + + A comma-separated list of + org.apache.hadoop.hbase.coprocessor.MasterObserver coprocessors that are + loaded by default on the active HMaster process. For any implemented + coprocessor methods, the listed classes will be called in order. After + implementing your own MasterObserver, just put it in HBase's classpath + and add the fully qualified class name here. + + + hbase.coprocessor.abortonerror false @@ -854,15 +931,6 @@ - dfs.support.append - true - Does HDFS allow appends to files? - This is an hdfs config. set in here so the hdfs client will do append support. - You must ensure that this config. is true serverside too when running hbase - (You will have to restart your cluster after setting it). - - - hbase.thrift.minWorkerThreads 16 @@ -930,37 +998,6 @@ - hbase.master.hfilecleaner.plugins - org.apache.hadoop.hbase.master.cleaner.TimeToLiveHFileCleaner - A comma-separated list of HFileCleanerDelegate invoked by - the HFileCleaner service. These HFiles cleaners are called in order, - so put the cleaner that prunes the most files in front. To - implement your own HFileCleanerDelegate, just put it in HBase's classpath - and add the fully qualified class name here. Always add the above - default log cleaners in the list as they will be overwritten in hbase-site.xml. - - - - hbase.regionserver.catalog.timeout - 600000 - Timeout value for the Catalog Janitor from the regionserver to META. - - - hbase.master.catalog.timeout - 600000 - Timeout value for the Catalog Janitor from the master to META. - - - hbase.config.read.zookeeper.config - false - - Set to true to allow HBaseConfiguration to read the - zoo.cfg file for ZooKeeper properties. Switching this to true - is not recommended, since the functionality of reading ZK - properties from a zoo.cfg file has been deprecated. - - - hbase.snapshot.enabled true @@ -968,39 +1005,6 @@ - hbase.rest.threads.max - 100 - - The maximum number of threads of the REST server thread pool. - Threads in the pool are reused to process REST requests. This - controls the maximum number of requests processed concurrently. - It may help to control the memory used by the REST server to - avoid OOM issues. If the thread pool is full, incoming requests - will be queued up and wait for some free threads. The default - is 100. - - - - hbase.rest.threads.min - 2 - - The minimum number of threads of the REST server thread pool. - The thread pool always has at least these number of threads so - the REST server is ready to serve incoming requests. The default - is 2. - - - - hbase.rpc.timeout - 60000 - - This is for the RPC layer to define how long HBase client applications - take for a remote call to time out. It uses pings to check connections - but will eventually throw a TimeoutException. - The default value is 60000ms(60s). - - - hbase.server.compactchecker.interval.multiplier 1000