You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@hbase.apache.org by GitBox <gi...@apache.org> on 2020/02/12 02:35:25 UTC

[GitHub] [hbase] bharathv opened a new pull request #1164: HBASE-23331: Documentation for HBASE-18095

bharathv opened a new pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164
 
 
   

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] ndimiduk commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
ndimiduk commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378611867
 
 

 ##########
 File path: src/main/asciidoc/_chapters/architecture.adoc
 ##########
 @@ -260,6 +260,73 @@ For region name, we only accept `byte[]` as the parameter type and it may be a f
 
 Information on non-Java clients and custom protocols is covered in <<external_apis>>
 
+[[client.masterregistry]]
+=== Master registry (new as of release 3.0.0)
+
+Client internally works with a _connection registry_ to fetch the metadata needed by connections.
+This connection registry implementation is responsible for fetching the following metadata.
+
+* Active master address
+* Current meta region(s) locations
+* Cluster ID (unique to this cluster)
+
+This information is needed as a part of various client operations like connection set up, scans,
+gets etc. Up until releases 2.x.y, the default connection registry is based on ZooKeeper as the
+source of truth and the the clients fetched the metadata from zookeeper znodes. As of release 3.0.0,
+the default implementation for connection registry has been switched  to a master based
+implementation. With this change, the clients now fetch the required metadata from master RPC end
+points directly. This change was done for the following reasons.
+
+* Reduce load on ZooKeeper since that is critical for cluster operation.
+* Holistic client timeout and retry configurations since the new registry brings all the client
+operations under HBase rpc framework.
+* Remove the ZooKeeper client dependency on HBase client library.
+
+This means that
+
+* At least a single active or stand by master is needed for cluster connection setup. Refer to
+<<master.runtime>> for more details.
+* Master can be in a critical path of read/write operations, especially if the client metadata cache
+is empty or stale.
+* There is higher connection load on the masters that before since the clients talk directly to
+HMasters instead of ZooKeeper ensemble`
+
+To reduce hot-spotting on a single master, all the masters (active & stand-by) expose the needed
+service to fetch the connection metadata. This lets the client connect to any master (not just active).
+
+==== RPC hedging
+
+This feature also implements an new RPC channel that can hedge requests to multiple masters. This
+lets the client make the same request to multiple servers and which ever responds first is returned
+back to the client and the other other in-flight requests are canceled. This improves the
+performance, especially when a subset of servers are under load. The hedging fan out size is
+configurable, meaning the number of requests that are hedged in a single attempt, using the
+configuration key _hbase.rpc.hedged.fanout_ in the client configuration. It defaults to 2. With this
+default, the RPCs are tried in batches of 2. The hedging policy is still primitive and does not
+adapt to any sort of live rpc performance metrics.
+
+==== Additional Notes
+
+* Clients hedge the requests in a randomized order to avoid hot-spotting a single server.
+* Cluster internal connections (master<->regionservers) still use ZooKeeper based connection
+registry.
+* Cluster internal state is still tracked in Zookeeper, hence ZK availability requirements are same
+as before.
+* Inter cluster replication still uses ZooKeeper beased connection registry to simplify configuration
+management.
+
+For more implementation details, please refer to the https://github.com/apache/hbase/tree/master/dev-support/design-docs[design doc] and
+https://issues.apache.org/jira/browse/HBASE-18095[HBASE-18095].
+
+'''
+NOTE: (Advanced) In case of any issues with the master based registry, use the following
+configuration to fallback to the ZooKeeper based connection registry implementation.
 
 Review comment:
   It's good that this is gated by a single config change.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] bharathv commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
bharathv commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378619263
 
 

 ##########
 File path: src/main/asciidoc/_chapters/configuration.adoc
 ##########
 @@ -563,38 +563,63 @@ Changes here will require a cluster restart for HBase to notice the change thoug
 
 If you are running HBase in standalone mode, you don't need to configure anything for your client to work provided that they are all on the same machine.
 
-Since the HBase Master may move around, clients bootstrap by looking to ZooKeeper for current critical locations.
-ZooKeeper is where all these values are kept.
-Thus clients require the location of the ZooKeeper ensemble before they can do anything else.
-Usually this ensemble location is kept out in the _hbase-site.xml_ and is picked up by the client from the `CLASSPATH`.
+Starting release 3.0.0, the default connection registry has been switched to a master based implementation. Refer to <<client.masterregistry>> for more details about
+what a connection registry is and implications of this change. Depending on your HBase version, following is the expected minimal client configuration.
 
-If you are configuring an IDE to run an HBase client, you should include the _conf/_ directory on your classpath so _hbase-site.xml_ settings can be found (or add _src/test/resources_ to pick up the hbase-site.xml used by tests).
+==== Up until 2.x.y releases
+In 2.x.y releases, the default connection registry was based on ZooKeeper as the source of truth. This means that the clients always looked up ZooKeeper znodes to fetch
+the required metadata. For example, if an active master crashed and the a new master is elected, clients looked up the master znode to fetch
+the active master address (similarly for meta locations). This meant that the clients needed to have access to ZooKeeper and need to now
 
 Review comment:
   Done.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] saintstack commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
saintstack commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378054046
 
 

 ##########
 File path: dev-support/design-docs/HBASE-18095-Zookeeper-less-client-connection-design.adoc
 ##########
 @@ -0,0 +1,112 @@
+////
+/**
+ *
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+////
+
+= HBASE-18095: Zookeeper-less client connection
+
+
+== Context
+Currently, Zookeeper (ZK) lies in the critical code path of connection init. To set up a connection to a given HBase cluster, client relies on the zookeeper quorum configured in the client hbase-site.xml and attempts to fetch the following information.
+
+* ClusterID
+* Active HMaster server name
+* Meta table region locations
+
+ZK is deemed the source of truth since other processes that maintain the cluster state persist the changes to this data into ZK. So it is an obvious place to look at for clients to fetch the latest cluster state.  However this comes with it’s own set of problems, some of them are below.
+
+* Timeouts and retry logic for ZK clients are managed separately from HBase configuration. This is more administration overhead for end users (example: multiple timeouts are to be configured for different types of RPCs. client->master, client->ZK etc.). This prevents HBase from having a single holistic timeout configuration that applies to any RPCs.
+* If there is any issue with ZK (like connection overload / timeouts), the entire HBase service appears frozen and there is little visibility into it.
+* Exposing zookeeper to all the clients can be risky since it can potentially be abused to DDOS.
+* Embedded ZK client is bundled with hbase client jar as a dependency (with it’s log spew :-]).
+
+== Goal
+
+We would like to remove this ZK dependency in the HBase client and instead have the clients query a preconfigured list of active and standby master host:port addresses. This brings all the client interactions with HBase under the same RPC framework that is holistically controlled by a set of hbase client configuration parameters. This also alleviates the pressure on ZK cluster which is critical from an operational standpoint as some core processes like replication, log splitting, master election etc. depend on it. The next section describes the kind of changes needed on both server and client side to support this behavior.
 
 Review comment:
   s/We would like to remove this/This feature removes the/

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] saintstack commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
saintstack commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378054583
 
 

 ##########
 File path: dev-support/design-docs/HBASE-18095-Zookeeper-less-client-connection-design.adoc
 ##########
 @@ -0,0 +1,112 @@
+////
+/**
+ *
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+////
+
+= HBASE-18095: Zookeeper-less client connection
+
+
+== Context
+Currently, Zookeeper (ZK) lies in the critical code path of connection init. To set up a connection to a given HBase cluster, client relies on the zookeeper quorum configured in the client hbase-site.xml and attempts to fetch the following information.
+
+* ClusterID
+* Active HMaster server name
+* Meta table region locations
+
+ZK is deemed the source of truth since other processes that maintain the cluster state persist the changes to this data into ZK. So it is an obvious place to look at for clients to fetch the latest cluster state.  However this comes with it’s own set of problems, some of them are below.
+
+* Timeouts and retry logic for ZK clients are managed separately from HBase configuration. This is more administration overhead for end users (example: multiple timeouts are to be configured for different types of RPCs. client->master, client->ZK etc.). This prevents HBase from having a single holistic timeout configuration that applies to any RPCs.
+* If there is any issue with ZK (like connection overload / timeouts), the entire HBase service appears frozen and there is little visibility into it.
+* Exposing zookeeper to all the clients can be risky since it can potentially be abused to DDOS.
+* Embedded ZK client is bundled with hbase client jar as a dependency (with it’s log spew :-]).
+
+== Goal
+
+We would like to remove this ZK dependency in the HBase client and instead have the clients query a preconfigured list of active and standby master host:port addresses. This brings all the client interactions with HBase under the same RPC framework that is holistically controlled by a set of hbase client configuration parameters. This also alleviates the pressure on ZK cluster which is critical from an operational standpoint as some core processes like replication, log splitting, master election etc. depend on it. The next section describes the kind of changes needed on both server and client side to support this behavior.
+
+== Proposed design
+
+As mentioned above, clients now get a pre configured list active and standby master addresses that they can query to fetch the meta information needed for connection setup. Something like,
+
+[source, xml]
+-----
+<property>
+  <name>hbase.masters</name>
+  <value>master1:16000,master2:16001,master3:16000</value>
+</property>
+-----
+
+Clients should be robust enough to handle configuration changes to this parameter since master hosts can change (added/removed) over time and not every client can afford a restart.
+
+One thing to note here is that having masters in the init/read/write path for clients means that
+
+* At Least one active/standby master is now needed for connection creation. Earlier this was not a requirement because the clients looked up the cluster ID from the relevant znode and init successfully. So, technically a master need not be around to create a connection to the cluster.
+* Masters are now active part of read write path in client life cycle under certain scenarios. If the client  cache of meta locations/active master is purged/stale, at least one master (active/stand-by) serving the latest information should exist. Earlier this information was served by ZK and clients look up the latest cluster ID/active master/meta locations from the relevant znodes and get going.
+* There is a higher connection load on the masters than before.
+* More state synchronization traffic (see below)
+
+End users should factor these requirements into their cluster deployment if they intend to use this feature.
+
+=== Server side changes
+
+Now that the master end points are considered as source of truth for clients, they should track the latest meta information for clusterID, active master and meta table locations. Since the clients can connect to any master end point (details below), all the masters (active/standby) now track all the relevant meta information. The idea is to implement an in-memory cache local to all the masters and it should keep up with changes to this metadata. This is tracked in the following jiras.
 
 Review comment:
   s/is to implement/is that we implement/

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] saintstack commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
saintstack commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378054145
 
 

 ##########
 File path: dev-support/design-docs/HBASE-18095-Zookeeper-less-client-connection-design.adoc
 ##########
 @@ -0,0 +1,112 @@
+////
+/**
+ *
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+////
+
+= HBASE-18095: Zookeeper-less client connection
+
+
+== Context
+Currently, Zookeeper (ZK) lies in the critical code path of connection init. To set up a connection to a given HBase cluster, client relies on the zookeeper quorum configured in the client hbase-site.xml and attempts to fetch the following information.
+
+* ClusterID
+* Active HMaster server name
+* Meta table region locations
+
+ZK is deemed the source of truth since other processes that maintain the cluster state persist the changes to this data into ZK. So it is an obvious place to look at for clients to fetch the latest cluster state.  However this comes with it’s own set of problems, some of them are below.
+
+* Timeouts and retry logic for ZK clients are managed separately from HBase configuration. This is more administration overhead for end users (example: multiple timeouts are to be configured for different types of RPCs. client->master, client->ZK etc.). This prevents HBase from having a single holistic timeout configuration that applies to any RPCs.
+* If there is any issue with ZK (like connection overload / timeouts), the entire HBase service appears frozen and there is little visibility into it.
+* Exposing zookeeper to all the clients can be risky since it can potentially be abused to DDOS.
+* Embedded ZK client is bundled with hbase client jar as a dependency (with it’s log spew :-]).
+
+== Goal
+
+We would like to remove this ZK dependency in the HBase client and instead have the clients query a preconfigured list of active and standby master host:port addresses. This brings all the client interactions with HBase under the same RPC framework that is holistically controlled by a set of hbase client configuration parameters. This also alleviates the pressure on ZK cluster which is critical from an operational standpoint as some core processes like replication, log splitting, master election etc. depend on it. The next section describes the kind of changes needed on both server and client side to support this behavior.
+
+== Proposed design
 
 Review comment:
   This is the refguide. Audience is operators. Also feature is 'done' so s/Proposed//

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] bharathv commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
bharathv commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378619073
 
 

 ##########
 File path: src/main/asciidoc/_chapters/architecture.adoc
 ##########
 @@ -260,6 +260,73 @@ For region name, we only accept `byte[]` as the parameter type and it may be a f
 
 Information on non-Java clients and custom protocols is covered in <<external_apis>>
 
+[[client.masterregistry]]
+=== Master registry (new as of release 3.0.0)
+
+Client internally works with a _connection registry_ to fetch the metadata needed by connections.
+This connection registry implementation is responsible for fetching the following metadata.
+
+* Active master address
+* Current meta region(s) locations
+* Cluster ID (unique to this cluster)
+
+This information is needed as a part of various client operations like connection set up, scans,
+gets etc. Up until releases 2.x.y, the default connection registry is based on ZooKeeper as the
+source of truth and the the clients fetched the metadata from zookeeper znodes. As of release 3.0.0,
+the default implementation for connection registry has been switched  to a master based
+implementation. With this change, the clients now fetch the required metadata from master RPC end
+points directly. This change was done for the following reasons.
+
+* Reduce load on ZooKeeper since that is critical for cluster operation.
+* Holistic client timeout and retry configurations since the new registry brings all the client
+operations under HBase rpc framework.
+* Remove the ZooKeeper client dependency on HBase client library.
+
+This means that
+
+* At least a single active or stand by master is needed for cluster connection setup. Refer to
+<<master.runtime>> for more details.
+* Master can be in a critical path of read/write operations, especially if the client metadata cache
+is empty or stale.
+* There is higher connection load on the masters that before since the clients talk directly to
+HMasters instead of ZooKeeper ensemble`
+
+To reduce hot-spotting on a single master, all the masters (active & stand-by) expose the needed
+service to fetch the connection metadata. This lets the client connect to any master (not just active).
+
+==== RPC hedging
+
+This feature also implements an new RPC channel that can hedge requests to multiple masters. This
+lets the client make the same request to multiple servers and which ever responds first is returned
+back to the client and the other other in-flight requests are canceled. This improves the
+performance, especially when a subset of servers are under load. The hedging fan out size is
+configurable, meaning the number of requests that are hedged in a single attempt, using the
+configuration key _hbase.rpc.hedged.fanout_ in the client configuration. It defaults to 2. With this
+default, the RPCs are tried in batches of 2. The hedging policy is still primitive and does not
+adapt to any sort of live rpc performance metrics.
+
+==== Additional Notes
+
+* Clients hedge the requests in a randomized order to avoid hot-spotting a single server.
+* Cluster internal connections (master<->regionservers) still use ZooKeeper based connection
+registry.
+* Cluster internal state is still tracked in Zookeeper, hence ZK availability requirements are same
+as before.
+* Inter cluster replication still uses ZooKeeper beased connection registry to simplify configuration
 
 Review comment:
   Done.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] bharathv commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
bharathv commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378619121
 
 

 ##########
 File path: src/main/asciidoc/_chapters/architecture.adoc
 ##########
 @@ -260,6 +260,73 @@ For region name, we only accept `byte[]` as the parameter type and it may be a f
 
 Information on non-Java clients and custom protocols is covered in <<external_apis>>
 
+[[client.masterregistry]]
+=== Master registry (new as of release 3.0.0)
+
+Client internally works with a _connection registry_ to fetch the metadata needed by connections.
+This connection registry implementation is responsible for fetching the following metadata.
+
+* Active master address
+* Current meta region(s) locations
+* Cluster ID (unique to this cluster)
+
+This information is needed as a part of various client operations like connection set up, scans,
+gets etc. Up until releases 2.x.y, the default connection registry is based on ZooKeeper as the
+source of truth and the the clients fetched the metadata from zookeeper znodes. As of release 3.0.0,
+the default implementation for connection registry has been switched  to a master based
+implementation. With this change, the clients now fetch the required metadata from master RPC end
+points directly. This change was done for the following reasons.
+
+* Reduce load on ZooKeeper since that is critical for cluster operation.
+* Holistic client timeout and retry configurations since the new registry brings all the client
+operations under HBase rpc framework.
+* Remove the ZooKeeper client dependency on HBase client library.
+
+This means that
+
+* At least a single active or stand by master is needed for cluster connection setup. Refer to
+<<master.runtime>> for more details.
+* Master can be in a critical path of read/write operations, especially if the client metadata cache
+is empty or stale.
+* There is higher connection load on the masters that before since the clients talk directly to
+HMasters instead of ZooKeeper ensemble`
+
+To reduce hot-spotting on a single master, all the masters (active & stand-by) expose the needed
+service to fetch the connection metadata. This lets the client connect to any master (not just active).
+
+==== RPC hedging
+
+This feature also implements an new RPC channel that can hedge requests to multiple masters. This
+lets the client make the same request to multiple servers and which ever responds first is returned
+back to the client and the other other in-flight requests are canceled. This improves the
+performance, especially when a subset of servers are under load. The hedging fan out size is
+configurable, meaning the number of requests that are hedged in a single attempt, using the
+configuration key _hbase.rpc.hedged.fanout_ in the client configuration. It defaults to 2. With this
+default, the RPCs are tried in batches of 2. The hedging policy is still primitive and does not
+adapt to any sort of live rpc performance metrics.
+
+==== Additional Notes
+
+* Clients hedge the requests in a randomized order to avoid hot-spotting a single server.
+* Cluster internal connections (master<->regionservers) still use ZooKeeper based connection
+registry.
+* Cluster internal state is still tracked in Zookeeper, hence ZK availability requirements are same
+as before.
+* Inter cluster replication still uses ZooKeeper beased connection registry to simplify configuration
+management.
+
+For more implementation details, please refer to the https://github.com/apache/hbase/tree/master/dev-support/design-docs[design doc] and
+https://issues.apache.org/jira/browse/HBASE-18095[HBASE-18095].
+
+'''
+NOTE: (Advanced) In case of any issues with the master based registry, use the following
+configuration to fallback to the ZooKeeper based connection registry implementation.
 
 Review comment:
   Yep.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] saintstack commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
saintstack commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378052401
 
 

 ##########
 File path: dev-support/design-docs/HBASE-18095-Zookeeper-less-client-connection-design.adoc
 ##########
 @@ -0,0 +1,112 @@
+////
+/**
+ *
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+////
+
+= HBASE-18095: Zookeeper-less client connection
+
+
+== Context
+Currently, Zookeeper (ZK) lies in the critical code path of connection init. To set up a connection to a given HBase cluster, client relies on the zookeeper quorum configured in the client hbase-site.xml and attempts to fetch the following information.
+
+* ClusterID
+* Active HMaster server name
+* Meta table region locations
+
+ZK is deemed the source of truth since other processes that maintain the cluster state persist the changes to this data into ZK. So it is an obvious place to look at for clients to fetch the latest cluster state.  However this comes with it’s own set of problems, some of them are below.
 
 Review comment:
   s/source of truth/store/ ?

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] ndimiduk commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
ndimiduk commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378610626
 
 

 ##########
 File path: dev-support/design-docs/HBASE-18095-Zookeeper-less-client-connection-design.adoc
 ##########
 @@ -0,0 +1,112 @@
+////
+/**
+ *
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+////
+
+= HBASE-18095: Zookeeper-less client connection
+
+
+== Context
+Currently, Zookeeper (ZK) lies in the critical code path of connection init. To set up a connection to a given HBase cluster, client relies on the zookeeper quorum configured in the client hbase-site.xml and attempts to fetch the following information.
+
+* ClusterID
+* Active HMaster server name
+* Meta table region locations
+
+ZK is deemed the source of truth since other processes that maintain the cluster state persist the changes to this data into ZK. So it is an obvious place to look at for clients to fetch the latest cluster state.  However this comes with it’s own set of problems, some of them are below.
+
+* Timeouts and retry logic for ZK clients are managed separately from HBase configuration. This is more administration overhead for end users (example: multiple timeouts are to be configured for different types of RPCs. client->master, client->ZK etc.). This prevents HBase from having a single holistic timeout configuration that applies to any RPCs.
+* If there is any issue with ZK (like connection overload / timeouts), the entire HBase service appears frozen and there is little visibility into it.
+* Exposing zookeeper to all the clients can be risky since it can potentially be abused to DDOS.
+* Embedded ZK client is bundled with hbase client jar as a dependency (with it’s log spew :-]).
+
+== Goal
+
+We would like to remove this ZK dependency in the HBase client and instead have the clients query a preconfigured list of active and standby master host:port addresses. This brings all the client interactions with HBase under the same RPC framework that is holistically controlled by a set of hbase client configuration parameters. This also alleviates the pressure on ZK cluster which is critical from an operational standpoint as some core processes like replication, log splitting, master election etc. depend on it. The next section describes the kind of changes needed on both server and client side to support this behavior.
+
+== Proposed design
+
+As mentioned above, clients now get a pre configured list active and standby master addresses that they can query to fetch the meta information needed for connection setup. Something like,
+
+[source, xml]
+-----
+<property>
+  <name>hbase.masters</name>
+  <value>master1:16000,master2:16001,master3:16000</value>
+</property>
+-----
+
+Clients should be robust enough to handle configuration changes to this parameter since master hosts can change (added/removed) over time and not every client can afford a restart.
+
+One thing to note here is that having masters in the init/read/write path for clients means that
+
+* At Least one active/standby master is now needed for connection creation. Earlier this was not a requirement because the clients looked up the cluster ID from the relevant znode and init successfully. So, technically a master need not be around to create a connection to the cluster.
+* Masters are now active part of read write path in client life cycle under certain scenarios. If the client  cache of meta locations/active master is purged/stale, at least one master (active/stand-by) serving the latest information should exist. Earlier this information was served by ZK and clients look up the latest cluster ID/active master/meta locations from the relevant znodes and get going.
 
 Review comment:
   I know we nit-picked over this language earlier. With this feature enabled, this change places the masters into the client lifecycle ~under _all_ scenarios~. It's simply the lifecycle. That involvement mirrors the current ZK involvement, which is to say that the masters are introduced into the path of service discovery that a client undertakes in order to locate the data it seeks.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] ndimiduk commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
ndimiduk commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378611201
 
 

 ##########
 File path: dev-support/design-docs/HBASE-18095-Zookeeper-less-client-connection-design.adoc
 ##########
 @@ -0,0 +1,112 @@
+////
+/**
+ *
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+////
+
+= HBASE-18095: Zookeeper-less client connection
+
+
+== Context
+Currently, Zookeeper (ZK) lies in the critical code path of connection init. To set up a connection to a given HBase cluster, client relies on the zookeeper quorum configured in the client hbase-site.xml and attempts to fetch the following information.
+
+* ClusterID
+* Active HMaster server name
+* Meta table region locations
+
+ZK is deemed the source of truth since other processes that maintain the cluster state persist the changes to this data into ZK. So it is an obvious place to look at for clients to fetch the latest cluster state.  However this comes with it’s own set of problems, some of them are below.
+
+* Timeouts and retry logic for ZK clients are managed separately from HBase configuration. This is more administration overhead for end users (example: multiple timeouts are to be configured for different types of RPCs. client->master, client->ZK etc.). This prevents HBase from having a single holistic timeout configuration that applies to any RPCs.
+* If there is any issue with ZK (like connection overload / timeouts), the entire HBase service appears frozen and there is little visibility into it.
+* Exposing zookeeper to all the clients can be risky since it can potentially be abused to DDOS.
+* Embedded ZK client is bundled with hbase client jar as a dependency (with it’s log spew :-]).
+
+== Goal
+
+We would like to remove this ZK dependency in the HBase client and instead have the clients query a preconfigured list of active and standby master host:port addresses. This brings all the client interactions with HBase under the same RPC framework that is holistically controlled by a set of hbase client configuration parameters. This also alleviates the pressure on ZK cluster which is critical from an operational standpoint as some core processes like replication, log splitting, master election etc. depend on it. The next section describes the kind of changes needed on both server and client side to support this behavior.
+
+== Proposed design
+
+As mentioned above, clients now get a pre configured list active and standby master addresses that they can query to fetch the meta information needed for connection setup. Something like,
+
+[source, xml]
+-----
+<property>
+  <name>hbase.masters</name>
+  <value>master1:16000,master2:16001,master3:16000</value>
+</property>
+-----
+
+Clients should be robust enough to handle configuration changes to this parameter since master hosts can change (added/removed) over time and not every client can afford a restart.
+
+One thing to note here is that having masters in the init/read/write path for clients means that
+
+* At Least one active/standby master is now needed for connection creation. Earlier this was not a requirement because the clients looked up the cluster ID from the relevant znode and init successfully. So, technically a master need not be around to create a connection to the cluster.
+* Masters are now active part of read write path in client life cycle under certain scenarios. If the client  cache of meta locations/active master is purged/stale, at least one master (active/stand-by) serving the latest information should exist. Earlier this information was served by ZK and clients look up the latest cluster ID/active master/meta locations from the relevant znodes and get going.
+* There is a higher connection load on the masters than before.
+* More state synchronization traffic (see below)
+
+End users should factor these requirements into their cluster deployment if they intend to use this feature.
+
+=== Server side changes
+
+Now that the master end points are considered as source of truth for clients, they should track the latest meta information for clusterID, active master and meta table locations. Since the clients can connect to any master end point (details below), all the masters (active/standby) now track all the relevant meta information. The idea is to implement an in-memory cache local to all the masters and it should keep up with changes to this metadata. This is tracked in the following jiras.
+
+* Clusterid tracking - https://issues.apache.org/jira/browse/HBASE-23257[HBASE-23257]
+* Active master tracking - https://issues.apache.org/jira/browse/HBASE-23275[HBASE-23275]
+* Meta location tracking - https://issues.apache.org/jira/browse/HBASE-23281[HBASE-23281]
+
+Masters and region servers (all cluster internal processes) still use ZK for cluster state coordination. Masters additionally cache this information in-memory and rely on ZK listeners and watchers to track the changes to them. New RPC endpoints are added to serve this information to the clients. Having an in-memory cache can speed up client lookups rather than fetching from ZK synchronously with the client lookup RPCs.
+
+=== Client side changes
+
+The proposal is to implement a new AsyncRegistry (https://issues.apache.org/jira/browse/HBASE-23305[HBASE-23305]) based on the master RPC endpoints discussed above. Few interesting optimizations on the client side as follows.
+
+* Each client randomly picks a master from the list of host:ports passed in the configuration. This avoids hotspotting on a single master.
+* Client can also do hedged lookups, meaning a single RPC to fetch meta information (say active master) can be sent to multiple masters and which ever returns first can be passed along to the caller. This can be done under a config flag since it comes with an additional RPC load. The default behavior is to randomly probe masters until the list is exhausted.
+* Callers are expected to cache the meta information in higher levels and only probe again if the cached information is stale (which is quite possible).
+
+One user noted that there are some clients that rely on cluster ID for delegation token based auth. So there is a proposal to expose it on an auth-less end point for delegation auth support..
 
 Review comment:
   Is there an issue filed for this? Might as well link it here. 

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] saintstack commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
saintstack commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378054451
 
 

 ##########
 File path: dev-support/design-docs/HBASE-18095-Zookeeper-less-client-connection-design.adoc
 ##########
 @@ -0,0 +1,112 @@
+////
+/**
+ *
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+////
+
+= HBASE-18095: Zookeeper-less client connection
+
+
+== Context
+Currently, Zookeeper (ZK) lies in the critical code path of connection init. To set up a connection to a given HBase cluster, client relies on the zookeeper quorum configured in the client hbase-site.xml and attempts to fetch the following information.
+
+* ClusterID
+* Active HMaster server name
+* Meta table region locations
+
+ZK is deemed the source of truth since other processes that maintain the cluster state persist the changes to this data into ZK. So it is an obvious place to look at for clients to fetch the latest cluster state.  However this comes with it’s own set of problems, some of them are below.
+
+* Timeouts and retry logic for ZK clients are managed separately from HBase configuration. This is more administration overhead for end users (example: multiple timeouts are to be configured for different types of RPCs. client->master, client->ZK etc.). This prevents HBase from having a single holistic timeout configuration that applies to any RPCs.
+* If there is any issue with ZK (like connection overload / timeouts), the entire HBase service appears frozen and there is little visibility into it.
+* Exposing zookeeper to all the clients can be risky since it can potentially be abused to DDOS.
+* Embedded ZK client is bundled with hbase client jar as a dependency (with it’s log spew :-]).
+
+== Goal
+
+We would like to remove this ZK dependency in the HBase client and instead have the clients query a preconfigured list of active and standby master host:port addresses. This brings all the client interactions with HBase under the same RPC framework that is holistically controlled by a set of hbase client configuration parameters. This also alleviates the pressure on ZK cluster which is critical from an operational standpoint as some core processes like replication, log splitting, master election etc. depend on it. The next section describes the kind of changes needed on both server and client side to support this behavior.
+
+== Proposed design
+
+As mentioned above, clients now get a pre configured list active and standby master addresses that they can query to fetch the meta information needed for connection setup. Something like,
+
+[source, xml]
+-----
+<property>
+  <name>hbase.masters</name>
+  <value>master1:16000,master2:16001,master3:16000</value>
+</property>
+-----
+
+Clients should be robust enough to handle configuration changes to this parameter since master hosts can change (added/removed) over time and not every client can afford a restart.
+
+One thing to note here is that having masters in the init/read/write path for clients means that
+
+* At Least one active/standby master is now needed for connection creation. Earlier this was not a requirement because the clients looked up the cluster ID from the relevant znode and init successfully. So, technically a master need not be around to create a connection to the cluster.
+* Masters are now active part of read write path in client life cycle under certain scenarios. If the client  cache of meta locations/active master is purged/stale, at least one master (active/stand-by) serving the latest information should exist. Earlier this information was served by ZK and clients look up the latest cluster ID/active master/meta locations from the relevant znodes and get going.
+* There is a higher connection load on the masters than before.
+* More state synchronization traffic (see below)
+
+End users should factor these requirements into their cluster deployment if they intend to use this feature.
+
+=== Server side changes
+
+Now that the master end points are considered as source of truth for clients, they should track the latest meta information for clusterID, active master and meta table locations. Since the clients can connect to any master end point (details below), all the masters (active/standby) now track all the relevant meta information. The idea is to implement an in-memory cache local to all the masters and it should keep up with changes to this metadata. This is tracked in the following jiras.
 
 Review comment:
   s/they should/they/

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] bharathv merged pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
bharathv merged pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164
 
 
   

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] bharathv commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
bharathv commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378618787
 
 

 ##########
 File path: dev-support/design-docs/HBASE-18095-Zookeeper-less-client-connection-design.adoc
 ##########
 @@ -0,0 +1,112 @@
+////
+/**
+ *
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+////
+
+= HBASE-18095: Zookeeper-less client connection
+
+
+== Context
+Currently, Zookeeper (ZK) lies in the critical code path of connection init. To set up a connection to a given HBase cluster, client relies on the zookeeper quorum configured in the client hbase-site.xml and attempts to fetch the following information.
+
+* ClusterID
+* Active HMaster server name
+* Meta table region locations
+
+ZK is deemed the source of truth since other processes that maintain the cluster state persist the changes to this data into ZK. So it is an obvious place to look at for clients to fetch the latest cluster state.  However this comes with it’s own set of problems, some of them are below.
+
+* Timeouts and retry logic for ZK clients are managed separately from HBase configuration. This is more administration overhead for end users (example: multiple timeouts are to be configured for different types of RPCs. client->master, client->ZK etc.). This prevents HBase from having a single holistic timeout configuration that applies to any RPCs.
+* If there is any issue with ZK (like connection overload / timeouts), the entire HBase service appears frozen and there is little visibility into it.
+* Exposing zookeeper to all the clients can be risky since it can potentially be abused to DDOS.
+* Embedded ZK client is bundled with hbase client jar as a dependency (with it’s log spew :-]).
+
+== Goal
+
+We would like to remove this ZK dependency in the HBase client and instead have the clients query a preconfigured list of active and standby master host:port addresses. This brings all the client interactions with HBase under the same RPC framework that is holistically controlled by a set of hbase client configuration parameters. This also alleviates the pressure on ZK cluster which is critical from an operational standpoint as some core processes like replication, log splitting, master election etc. depend on it. The next section describes the kind of changes needed on both server and client side to support this behavior.
+
+== Proposed design
+
+As mentioned above, clients now get a pre configured list active and standby master addresses that they can query to fetch the meta information needed for connection setup. Something like,
+
+[source, xml]
+-----
+<property>
+  <name>hbase.masters</name>
+  <value>master1:16000,master2:16001,master3:16000</value>
+</property>
+-----
+
+Clients should be robust enough to handle configuration changes to this parameter since master hosts can change (added/removed) over time and not every client can afford a restart.
+
+One thing to note here is that having masters in the init/read/write path for clients means that
+
+* At Least one active/standby master is now needed for connection creation. Earlier this was not a requirement because the clients looked up the cluster ID from the relevant znode and init successfully. So, technically a master need not be around to create a connection to the cluster.
+* Masters are now active part of read write path in client life cycle under certain scenarios. If the client  cache of meta locations/active master is purged/stale, at least one master (active/stand-by) serving the latest information should exist. Earlier this information was served by ZK and clients look up the latest cluster ID/active master/meta locations from the relevant znodes and get going.
+* There is a higher connection load on the masters than before.
+* More state synchronization traffic (see below)
+
+End users should factor these requirements into their cluster deployment if they intend to use this feature.
+
+=== Server side changes
+
+Now that the master end points are considered as source of truth for clients, they should track the latest meta information for clusterID, active master and meta table locations. Since the clients can connect to any master end point (details below), all the masters (active/standby) now track all the relevant meta information. The idea is to implement an in-memory cache local to all the masters and it should keep up with changes to this metadata. This is tracked in the following jiras.
+
+* Clusterid tracking - https://issues.apache.org/jira/browse/HBASE-23257[HBASE-23257]
+* Active master tracking - https://issues.apache.org/jira/browse/HBASE-23275[HBASE-23275]
+* Meta location tracking - https://issues.apache.org/jira/browse/HBASE-23281[HBASE-23281]
+
+Masters and region servers (all cluster internal processes) still use ZK for cluster state coordination. Masters additionally cache this information in-memory and rely on ZK listeners and watchers to track the changes to them. New RPC endpoints are added to serve this information to the clients. Having an in-memory cache can speed up client lookups rather than fetching from ZK synchronously with the client lookup RPCs.
+
+=== Client side changes
+
+The proposal is to implement a new AsyncRegistry (https://issues.apache.org/jira/browse/HBASE-23305[HBASE-23305]) based on the master RPC endpoints discussed above. Few interesting optimizations on the client side as follows.
+
+* Each client randomly picks a master from the list of host:ports passed in the configuration. This avoids hotspotting on a single master.
+* Client can also do hedged lookups, meaning a single RPC to fetch meta information (say active master) can be sent to multiple masters and which ever returns first can be passed along to the caller. This can be done under a config flag since it comes with an additional RPC load. The default behavior is to randomly probe masters until the list is exhausted.
+* Callers are expected to cache the meta information in higher levels and only probe again if the cached information is stale (which is quite possible).
+
+One user noted that there are some clients that rely on cluster ID for delegation token based auth. So there is a proposal to expose it on an auth-less end point for delegation auth support..
 
 Review comment:
   HBASE-23330, yes that was fixed too.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] Apache-HBase commented on issue #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
Apache-HBase commented on issue #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#issuecomment-584998819
 
 
   :confetti_ball: **+1 overall**
   
   
   
   
   
   
   | Vote | Subsystem | Runtime | Comment |
   |:----:|----------:|--------:|:--------|
   | +0 :ok: |  reexec  |   0m 31s |  Docker mode activated.  |
   ||| _ Prechecks _ |
   | +1 :green_heart: |  dupname  |   0m  0s |  No case conflicting files found.  |
   | +1 :green_heart: |  @author  |   0m  0s |  The patch does not contain any @author tags.  |
   ||| _ HBASE-18095/client-locate-meta-no-zookeeper Compile Tests _ |
   | +1 :green_heart: |  mvninstall  |   5m 32s |  HBASE-18095/client-locate-meta-no-zookeeper passed  |
   | +0 :ok: |  refguide  |   4m 46s |  branch has no errors when building the reference guide. See footer for rendered docs, which you should manually inspect.  |
   ||| _ Patch Compile Tests _ |
   | +1 :green_heart: |  mvninstall  |   5m  4s |  the patch passed  |
   | +1 :green_heart: |  whitespace  |   0m  0s |  The patch has no whitespace issues.  |
   | +0 :ok: |  refguide  |   4m 50s |  patch has no errors when building the reference guide. See footer for rendered docs, which you should manually inspect.  |
   ||| _ Other Tests _ |
   | +1 :green_heart: |  asflicense  |   0m 19s |  The patch does not generate ASF License warnings.  |
   |  |   |  22m 12s |   |
   
   
   | Subsystem | Report/Notes |
   |----------:|:-------------|
   | Docker | Client=19.03.5 Server=19.03.5 base: https://builds.apache.org/job/HBase-PreCommit-GitHub-PR/job/PR-1164/1/artifact/out/Dockerfile |
   | GITHUB PR | https://github.com/apache/hbase/pull/1164 |
   | JIRA Issue | HBASE-23331 |
   | Optional Tests | dupname asflicense refguide |
   | uname | Linux 40c0e195bda9 4.15.0-58-generic #64-Ubuntu SMP Tue Aug 6 11:12:41 UTC 2019 x86_64 GNU/Linux |
   | Build tool | maven |
   | Personality | /home/jenkins/jenkins-slave/workspace/Base-PreCommit-GitHub-PR_PR-1164/out/precommit/personality/provided.sh |
   | git revision | HBASE-18095/client-locate-meta-no-zookeeper / 2eead6c9ff |
   | refguide | https://builds.apache.org/job/HBase-PreCommit-GitHub-PR/job/PR-1164/1/artifact/out/branch-site/book.html |
   | refguide | https://builds.apache.org/job/HBase-PreCommit-GitHub-PR/job/PR-1164/1/artifact/out/patch-site/book.html |
   | Max. process+thread count | 96 (vs. ulimit of 10000) |
   | modules | C: . U: . |
   | Console output | https://builds.apache.org/job/HBase-PreCommit-GitHub-PR/job/PR-1164/1/console |
   | versions | git=2.11.0 maven=2018-06-17T18:33:14Z) |
   | Powered by | Apache Yetus 0.11.1 https://yetus.apache.org |
   
   
   This message was automatically generated.
   
   

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] saintstack commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
saintstack commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378053858
 
 

 ##########
 File path: dev-support/design-docs/HBASE-18095-Zookeeper-less-client-connection-design.adoc
 ##########
 @@ -0,0 +1,112 @@
+////
+/**
+ *
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+////
+
+= HBASE-18095: Zookeeper-less client connection
+
+
+== Context
+Currently, Zookeeper (ZK) lies in the critical code path of connection init. To set up a connection to a given HBase cluster, client relies on the zookeeper quorum configured in the client hbase-site.xml and attempts to fetch the following information.
+
+* ClusterID
+* Active HMaster server name
+* Meta table region locations
+
+ZK is deemed the source of truth since other processes that maintain the cluster state persist the changes to this data into ZK. So it is an obvious place to look at for clients to fetch the latest cluster state.  However this comes with it’s own set of problems, some of them are below.
+
+* Timeouts and retry logic for ZK clients are managed separately from HBase configuration. This is more administration overhead for end users (example: multiple timeouts are to be configured for different types of RPCs. client->master, client->ZK etc.). This prevents HBase from having a single holistic timeout configuration that applies to any RPCs.
+* If there is any issue with ZK (like connection overload / timeouts), the entire HBase service appears frozen and there is little visibility into it.
+* Exposing zookeeper to all the clients can be risky since it can potentially be abused to DDOS.
+* Embedded ZK client is bundled with hbase client jar as a dependency (with it’s log spew :-]).
+
+== Goal
 
 Review comment:
   s/Goal/Description/

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] saintstack commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
saintstack commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378054311
 
 

 ##########
 File path: dev-support/design-docs/HBASE-18095-Zookeeper-less-client-connection-design.adoc
 ##########
 @@ -0,0 +1,112 @@
+////
+/**
+ *
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+////
+
+= HBASE-18095: Zookeeper-less client connection
+
+
+== Context
+Currently, Zookeeper (ZK) lies in the critical code path of connection init. To set up a connection to a given HBase cluster, client relies on the zookeeper quorum configured in the client hbase-site.xml and attempts to fetch the following information.
+
+* ClusterID
+* Active HMaster server name
+* Meta table region locations
+
+ZK is deemed the source of truth since other processes that maintain the cluster state persist the changes to this data into ZK. So it is an obvious place to look at for clients to fetch the latest cluster state.  However this comes with it’s own set of problems, some of them are below.
+
+* Timeouts and retry logic for ZK clients are managed separately from HBase configuration. This is more administration overhead for end users (example: multiple timeouts are to be configured for different types of RPCs. client->master, client->ZK etc.). This prevents HBase from having a single holistic timeout configuration that applies to any RPCs.
+* If there is any issue with ZK (like connection overload / timeouts), the entire HBase service appears frozen and there is little visibility into it.
+* Exposing zookeeper to all the clients can be risky since it can potentially be abused to DDOS.
+* Embedded ZK client is bundled with hbase client jar as a dependency (with it’s log spew :-]).
+
+== Goal
+
+We would like to remove this ZK dependency in the HBase client and instead have the clients query a preconfigured list of active and standby master host:port addresses. This brings all the client interactions with HBase under the same RPC framework that is holistically controlled by a set of hbase client configuration parameters. This also alleviates the pressure on ZK cluster which is critical from an operational standpoint as some core processes like replication, log splitting, master election etc. depend on it. The next section describes the kind of changes needed on both server and client side to support this behavior.
+
+== Proposed design
+
+As mentioned above, clients now get a pre configured list active and standby master addresses that they can query to fetch the meta information needed for connection setup. Something like,
+
+[source, xml]
+-----
+<property>
+  <name>hbase.masters</name>
+  <value>master1:16000,master2:16001,master3:16000</value>
+</property>
+-----
+
+Clients should be robust enough to handle configuration changes to this parameter since master hosts can change (added/removed) over time and not every client can afford a restart.
+
+One thing to note here is that having masters in the init/read/write path for clients means that
+
+* At Least one active/standby master is now needed for connection creation. Earlier this was not a requirement because the clients looked up the cluster ID from the relevant znode and init successfully. So, technically a master need not be around to create a connection to the cluster.
+* Masters are now active part of read write path in client life cycle under certain scenarios. If the client  cache of meta locations/active master is purged/stale, at least one master (active/stand-by) serving the latest information should exist. Earlier this information was served by ZK and clients look up the latest cluster ID/active master/meta locations from the relevant znodes and get going.
+* There is a higher connection load on the masters than before.
+* More state synchronization traffic (see below)
 
 Review comment:
   Good

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] saintstack commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
saintstack commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378052549
 
 

 ##########
 File path: dev-support/design-docs/HBASE-18095-Zookeeper-less-client-connection-design.adoc
 ##########
 @@ -0,0 +1,112 @@
+////
+/**
+ *
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+////
+
+= HBASE-18095: Zookeeper-less client connection
+
+
+== Context
+Currently, Zookeeper (ZK) lies in the critical code path of connection init. To set up a connection to a given HBase cluster, client relies on the zookeeper quorum configured in the client hbase-site.xml and attempts to fetch the following information.
+
+* ClusterID
+* Active HMaster server name
+* Meta table region locations
+
+ZK is deemed the source of truth since other processes that maintain the cluster state persist the changes to this data into ZK. So it is an obvious place to look at for clients to fetch the latest cluster state.  However this comes with it’s own set of problems, some of them are below.
+
+* Timeouts and retry logic for ZK clients are managed separately from HBase configuration. This is more administration overhead for end users (example: multiple timeouts are to be configured for different types of RPCs. client->master, client->ZK etc.). This prevents HBase from having a single holistic timeout configuration that applies to any RPCs.
 
 Review comment:
   Good

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] ndimiduk commented on issue #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
ndimiduk commented on issue #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#issuecomment-585510196
 
 
   Oh, I see you merged the PR already. The JIRA was still open, so I assumed it was not in yet.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] saintstack commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
saintstack commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378054663
 
 

 ##########
 File path: dev-support/design-docs/HBASE-18095-Zookeeper-less-client-connection-design.adoc
 ##########
 @@ -0,0 +1,112 @@
+////
+/**
+ *
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+////
+
+= HBASE-18095: Zookeeper-less client connection
+
+
+== Context
+Currently, Zookeeper (ZK) lies in the critical code path of connection init. To set up a connection to a given HBase cluster, client relies on the zookeeper quorum configured in the client hbase-site.xml and attempts to fetch the following information.
+
+* ClusterID
+* Active HMaster server name
+* Meta table region locations
+
+ZK is deemed the source of truth since other processes that maintain the cluster state persist the changes to this data into ZK. So it is an obvious place to look at for clients to fetch the latest cluster state.  However this comes with it’s own set of problems, some of them are below.
+
+* Timeouts and retry logic for ZK clients are managed separately from HBase configuration. This is more administration overhead for end users (example: multiple timeouts are to be configured for different types of RPCs. client->master, client->ZK etc.). This prevents HBase from having a single holistic timeout configuration that applies to any RPCs.
+* If there is any issue with ZK (like connection overload / timeouts), the entire HBase service appears frozen and there is little visibility into it.
+* Exposing zookeeper to all the clients can be risky since it can potentially be abused to DDOS.
+* Embedded ZK client is bundled with hbase client jar as a dependency (with it’s log spew :-]).
+
+== Goal
+
+We would like to remove this ZK dependency in the HBase client and instead have the clients query a preconfigured list of active and standby master host:port addresses. This brings all the client interactions with HBase under the same RPC framework that is holistically controlled by a set of hbase client configuration parameters. This also alleviates the pressure on ZK cluster which is critical from an operational standpoint as some core processes like replication, log splitting, master election etc. depend on it. The next section describes the kind of changes needed on both server and client side to support this behavior.
+
+== Proposed design
+
+As mentioned above, clients now get a pre configured list active and standby master addresses that they can query to fetch the meta information needed for connection setup. Something like,
+
+[source, xml]
+-----
+<property>
+  <name>hbase.masters</name>
+  <value>master1:16000,master2:16001,master3:16000</value>
+</property>
+-----
+
+Clients should be robust enough to handle configuration changes to this parameter since master hosts can change (added/removed) over time and not every client can afford a restart.
+
+One thing to note here is that having masters in the init/read/write path for clients means that
+
+* At Least one active/standby master is now needed for connection creation. Earlier this was not a requirement because the clients looked up the cluster ID from the relevant znode and init successfully. So, technically a master need not be around to create a connection to the cluster.
+* Masters are now active part of read write path in client life cycle under certain scenarios. If the client  cache of meta locations/active master is purged/stale, at least one master (active/stand-by) serving the latest information should exist. Earlier this information was served by ZK and clients look up the latest cluster ID/active master/meta locations from the relevant znodes and get going.
+* There is a higher connection load on the masters than before.
+* More state synchronization traffic (see below)
+
+End users should factor these requirements into their cluster deployment if they intend to use this feature.
+
+=== Server side changes
+
+Now that the master end points are considered as source of truth for clients, they should track the latest meta information for clusterID, active master and meta table locations. Since the clients can connect to any master end point (details below), all the masters (active/standby) now track all the relevant meta information. The idea is to implement an in-memory cache local to all the masters and it should keep up with changes to this metadata. This is tracked in the following jiras.
+
+* Clusterid tracking - https://issues.apache.org/jira/browse/HBASE-23257[HBASE-23257]
+* Active master tracking - https://issues.apache.org/jira/browse/HBASE-23275[HBASE-23275]
+* Meta location tracking - https://issues.apache.org/jira/browse/HBASE-23281[HBASE-23281]
 
 Review comment:
   TMI No need of these JIRA listings.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] bharathv commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
bharathv commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378623912
 
 

 ##########
 File path: dev-support/design-docs/HBASE-18095-Zookeeper-less-client-connection-design.adoc
 ##########
 @@ -0,0 +1,112 @@
+////
+/**
+ *
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+////
+
+= HBASE-18095: Zookeeper-less client connection
+
+
+== Context
+Currently, Zookeeper (ZK) lies in the critical code path of connection init. To set up a connection to a given HBase cluster, client relies on the zookeeper quorum configured in the client hbase-site.xml and attempts to fetch the following information.
+
+* ClusterID
+* Active HMaster server name
+* Meta table region locations
+
+ZK is deemed the source of truth since other processes that maintain the cluster state persist the changes to this data into ZK. So it is an obvious place to look at for clients to fetch the latest cluster state.  However this comes with it’s own set of problems, some of them are below.
+
+* Timeouts and retry logic for ZK clients are managed separately from HBase configuration. This is more administration overhead for end users (example: multiple timeouts are to be configured for different types of RPCs. client->master, client->ZK etc.). This prevents HBase from having a single holistic timeout configuration that applies to any RPCs.
+* If there is any issue with ZK (like connection overload / timeouts), the entire HBase service appears frozen and there is little visibility into it.
+* Exposing zookeeper to all the clients can be risky since it can potentially be abused to DDOS.
+* Embedded ZK client is bundled with hbase client jar as a dependency (with it’s log spew :-]).
+
+== Goal
+
+We would like to remove this ZK dependency in the HBase client and instead have the clients query a preconfigured list of active and standby master host:port addresses. This brings all the client interactions with HBase under the same RPC framework that is holistically controlled by a set of hbase client configuration parameters. This also alleviates the pressure on ZK cluster which is critical from an operational standpoint as some core processes like replication, log splitting, master election etc. depend on it. The next section describes the kind of changes needed on both server and client side to support this behavior.
+
+== Proposed design
+
+As mentioned above, clients now get a pre configured list active and standby master addresses that they can query to fetch the meta information needed for connection setup. Something like,
+
+[source, xml]
+-----
+<property>
+  <name>hbase.masters</name>
+  <value>master1:16000,master2:16001,master3:16000</value>
+</property>
+-----
+
+Clients should be robust enough to handle configuration changes to this parameter since master hosts can change (added/removed) over time and not every client can afford a restart.
+
+One thing to note here is that having masters in the init/read/write path for clients means that
+
+* At Least one active/standby master is now needed for connection creation. Earlier this was not a requirement because the clients looked up the cluster ID from the relevant znode and init successfully. So, technically a master need not be around to create a connection to the cluster.
+* Masters are now active part of read write path in client life cycle under certain scenarios. If the client  cache of meta locations/active master is purged/stale, at least one master (active/stand-by) serving the latest information should exist. Earlier this information was served by ZK and clients look up the latest cluster ID/active master/meta locations from the relevant znodes and get going.
 
 Review comment:
   Fixed.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] ndimiduk commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
ndimiduk commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378611425
 
 

 ##########
 File path: src/main/asciidoc/_chapters/architecture.adoc
 ##########
 @@ -260,6 +260,73 @@ For region name, we only accept `byte[]` as the parameter type and it may be a f
 
 Information on non-Java clients and custom protocols is covered in <<external_apis>>
 
+[[client.masterregistry]]
+=== Master registry (new as of release 3.0.0)
 
 Review comment:
   New as of 2.3.0, right? I think you'll want to update this throughout the doc.

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] ndimiduk commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
ndimiduk commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378612292
 
 

 ##########
 File path: src/main/asciidoc/_chapters/configuration.adoc
 ##########
 @@ -563,38 +563,63 @@ Changes here will require a cluster restart for HBase to notice the change thoug
 
 If you are running HBase in standalone mode, you don't need to configure anything for your client to work provided that they are all on the same machine.
 
-Since the HBase Master may move around, clients bootstrap by looking to ZooKeeper for current critical locations.
-ZooKeeper is where all these values are kept.
-Thus clients require the location of the ZooKeeper ensemble before they can do anything else.
-Usually this ensemble location is kept out in the _hbase-site.xml_ and is picked up by the client from the `CLASSPATH`.
+Starting release 3.0.0, the default connection registry has been switched to a master based implementation. Refer to <<client.masterregistry>> for more details about
+what a connection registry is and implications of this change. Depending on your HBase version, following is the expected minimal client configuration.
 
-If you are configuring an IDE to run an HBase client, you should include the _conf/_ directory on your classpath so _hbase-site.xml_ settings can be found (or add _src/test/resources_ to pick up the hbase-site.xml used by tests).
+==== Up until 2.x.y releases
+In 2.x.y releases, the default connection registry was based on ZooKeeper as the source of truth. This means that the clients always looked up ZooKeeper znodes to fetch
+the required metadata. For example, if an active master crashed and the a new master is elected, clients looked up the master znode to fetch
+the active master address (similarly for meta locations). This meant that the clients needed to have access to ZooKeeper and need to now
 
 Review comment:
   a/now/know/

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] ndimiduk commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
ndimiduk commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378612539
 
 

 ##########
 File path: src/main/asciidoc/_chapters/configuration.adoc
 ##########
 @@ -563,38 +563,63 @@ Changes here will require a cluster restart for HBase to notice the change thoug
 
 If you are running HBase in standalone mode, you don't need to configure anything for your client to work provided that they are all on the same machine.
 
-Since the HBase Master may move around, clients bootstrap by looking to ZooKeeper for current critical locations.
-ZooKeeper is where all these values are kept.
-Thus clients require the location of the ZooKeeper ensemble before they can do anything else.
-Usually this ensemble location is kept out in the _hbase-site.xml_ and is picked up by the client from the `CLASSPATH`.
+Starting release 3.0.0, the default connection registry has been switched to a master based implementation. Refer to <<client.masterregistry>> for more details about
+what a connection registry is and implications of this change. Depending on your HBase version, following is the expected minimal client configuration.
 
-If you are configuring an IDE to run an HBase client, you should include the _conf/_ directory on your classpath so _hbase-site.xml_ settings can be found (or add _src/test/resources_ to pick up the hbase-site.xml used by tests).
+==== Up until 2.x.y releases
+In 2.x.y releases, the default connection registry was based on ZooKeeper as the source of truth. This means that the clients always looked up ZooKeeper znodes to fetch
+the required metadata. For example, if an active master crashed and the a new master is elected, clients looked up the master znode to fetch
+the active master address (similarly for meta locations). This meant that the clients needed to have access to ZooKeeper and need to now
+the ZooKeeper ensemble information before they can do anything. This can be configured in the client configuration xml as follows:
 
-For Java applications using Maven, including the hbase-shaded-client module is the recommended dependency when connecting to a cluster:
 [source,xml]
 ----
-<dependency>
-  <groupId>org.apache.hbase</groupId>
-  <artifactId>hbase-shaded-client</artifactId>
-  <version>2.0.0</version>
-</dependency>
+<?xml version="1.0"?>
+<?xml-stylesheet type="text/xsl" href="configuration.xsl"?>
+<configuration>
+  <property>
+    <name>hbase.zookeeper.quorum</name>
+    <value>example1,example2,example3</value>
+    <description> Zookeeper ensemble information</description>
+  </property>
+</configuration>
 ----
 
-A basic example _hbase-site.xml_ for client only may look as follows:
+==== Starting 3.0.0 release
+
+The default implementation was switched to a master based connection registry. With this implementation, clients always contact the active or
+stand-by master RPC end points to fetch the the connection registry information. This means that the clients should have access to the list of active and master
+end points before they can do anything. This can be configured in the client configuration xml as follows:
+
 [source,xml]
 ----
 <?xml version="1.0"?>
 <?xml-stylesheet type="text/xsl" href="configuration.xsl"?>
 <configuration>
   <property>
-    <name>hbase.zookeeper.quorum</name>
+    <name>hbase.masters</name>
     <value>example1,example2,example3</value>
-    <description>The directory shared by region servers.
-    </description>
+    <description>List of master rpc end points for the hbase cluster.</description>
   </property>
 </configuration>
 ----
 
+The configuration value for _hbase.masters_ is a comma separated list of _host:port_ values. If no port value is specified, the default of _16000_ is assumed.
+
+Usually this configuration is kept out in the _hbase-site.xml_ and is picked up by the client from the `CLASSPATH`.
+
+If you are configuring an IDE to run an HBase client, you should include the _conf/_ directory on your classpath so _hbase-site.xml_ settings can be found (or add _src/test/resources_ to pick up the hbase-site.xml used by tests).
+
+For Java applications using Maven, including the hbase-shaded-client module is the recommended dependency when connecting to a cluster:
+[source,xml]
 
 Review comment:
   nit: is dependency information relevant in the configuration section?

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] ndimiduk commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
ndimiduk commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378611722
 
 

 ##########
 File path: src/main/asciidoc/_chapters/architecture.adoc
 ##########
 @@ -260,6 +260,73 @@ For region name, we only accept `byte[]` as the parameter type and it may be a f
 
 Information on non-Java clients and custom protocols is covered in <<external_apis>>
 
+[[client.masterregistry]]
+=== Master registry (new as of release 3.0.0)
+
+Client internally works with a _connection registry_ to fetch the metadata needed by connections.
+This connection registry implementation is responsible for fetching the following metadata.
+
+* Active master address
+* Current meta region(s) locations
+* Cluster ID (unique to this cluster)
+
+This information is needed as a part of various client operations like connection set up, scans,
+gets etc. Up until releases 2.x.y, the default connection registry is based on ZooKeeper as the
+source of truth and the the clients fetched the metadata from zookeeper znodes. As of release 3.0.0,
+the default implementation for connection registry has been switched  to a master based
+implementation. With this change, the clients now fetch the required metadata from master RPC end
+points directly. This change was done for the following reasons.
+
+* Reduce load on ZooKeeper since that is critical for cluster operation.
+* Holistic client timeout and retry configurations since the new registry brings all the client
+operations under HBase rpc framework.
+* Remove the ZooKeeper client dependency on HBase client library.
+
+This means that
+
+* At least a single active or stand by master is needed for cluster connection setup. Refer to
+<<master.runtime>> for more details.
+* Master can be in a critical path of read/write operations, especially if the client metadata cache
+is empty or stale.
+* There is higher connection load on the masters that before since the clients talk directly to
+HMasters instead of ZooKeeper ensemble`
+
+To reduce hot-spotting on a single master, all the masters (active & stand-by) expose the needed
+service to fetch the connection metadata. This lets the client connect to any master (not just active).
+
+==== RPC hedging
+
+This feature also implements an new RPC channel that can hedge requests to multiple masters. This
+lets the client make the same request to multiple servers and which ever responds first is returned
+back to the client and the other other in-flight requests are canceled. This improves the
+performance, especially when a subset of servers are under load. The hedging fan out size is
+configurable, meaning the number of requests that are hedged in a single attempt, using the
+configuration key _hbase.rpc.hedged.fanout_ in the client configuration. It defaults to 2. With this
+default, the RPCs are tried in batches of 2. The hedging policy is still primitive and does not
+adapt to any sort of live rpc performance metrics.
+
+==== Additional Notes
+
+* Clients hedge the requests in a randomized order to avoid hot-spotting a single server.
+* Cluster internal connections (master<->regionservers) still use ZooKeeper based connection
+registry.
+* Cluster internal state is still tracked in Zookeeper, hence ZK availability requirements are same
+as before.
+* Inter cluster replication still uses ZooKeeper beased connection registry to simplify configuration
 
 Review comment:
   s/beased/based/

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] bharathv commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
bharathv commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378618988
 
 

 ##########
 File path: src/main/asciidoc/_chapters/architecture.adoc
 ##########
 @@ -260,6 +260,73 @@ For region name, we only accept `byte[]` as the parameter type and it may be a f
 
 Information on non-Java clients and custom protocols is covered in <<external_apis>>
 
+[[client.masterregistry]]
+=== Master registry (new as of release 3.0.0)
 
 Review comment:
   I wanted to update the docs once it actually makes it to the 2.3.0...that ok? (didn't want to do that without committing to the branch).

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services

[GitHub] [hbase] bharathv commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095

Posted by GitBox <gi...@apache.org>.
bharathv commented on a change in pull request #1164: HBASE-23331: Documentation for HBASE-18095
URL: https://github.com/apache/hbase/pull/1164#discussion_r378620758
 
 

 ##########
 File path: src/main/asciidoc/_chapters/configuration.adoc
 ##########
 @@ -563,38 +563,63 @@ Changes here will require a cluster restart for HBase to notice the change thoug
 
 If you are running HBase in standalone mode, you don't need to configure anything for your client to work provided that they are all on the same machine.
 
-Since the HBase Master may move around, clients bootstrap by looking to ZooKeeper for current critical locations.
-ZooKeeper is where all these values are kept.
-Thus clients require the location of the ZooKeeper ensemble before they can do anything else.
-Usually this ensemble location is kept out in the _hbase-site.xml_ and is picked up by the client from the `CLASSPATH`.
+Starting release 3.0.0, the default connection registry has been switched to a master based implementation. Refer to <<client.masterregistry>> for more details about
+what a connection registry is and implications of this change. Depending on your HBase version, following is the expected minimal client configuration.
 
-If you are configuring an IDE to run an HBase client, you should include the _conf/_ directory on your classpath so _hbase-site.xml_ settings can be found (or add _src/test/resources_ to pick up the hbase-site.xml used by tests).
+==== Up until 2.x.y releases
+In 2.x.y releases, the default connection registry was based on ZooKeeper as the source of truth. This means that the clients always looked up ZooKeeper znodes to fetch
+the required metadata. For example, if an active master crashed and the a new master is elected, clients looked up the master znode to fetch
+the active master address (similarly for meta locations). This meant that the clients needed to have access to ZooKeeper and need to now
+the ZooKeeper ensemble information before they can do anything. This can be configured in the client configuration xml as follows:
 
-For Java applications using Maven, including the hbase-shaded-client module is the recommended dependency when connecting to a cluster:
 [source,xml]
 ----
-<dependency>
-  <groupId>org.apache.hbase</groupId>
-  <artifactId>hbase-shaded-client</artifactId>
-  <version>2.0.0</version>
-</dependency>
+<?xml version="1.0"?>
+<?xml-stylesheet type="text/xsl" href="configuration.xsl"?>
+<configuration>
+  <property>
+    <name>hbase.zookeeper.quorum</name>
+    <value>example1,example2,example3</value>
+    <description> Zookeeper ensemble information</description>
+  </property>
+</configuration>
 ----
 
-A basic example _hbase-site.xml_ for client only may look as follows:
+==== Starting 3.0.0 release
+
+The default implementation was switched to a master based connection registry. With this implementation, clients always contact the active or
+stand-by master RPC end points to fetch the the connection registry information. This means that the clients should have access to the list of active and master
+end points before they can do anything. This can be configured in the client configuration xml as follows:
+
 [source,xml]
 ----
 <?xml version="1.0"?>
 <?xml-stylesheet type="text/xsl" href="configuration.xsl"?>
 <configuration>
   <property>
-    <name>hbase.zookeeper.quorum</name>
+    <name>hbase.masters</name>
     <value>example1,example2,example3</value>
-    <description>The directory shared by region servers.
-    </description>
+    <description>List of master rpc end points for the hbase cluster.</description>
   </property>
 </configuration>
 ----
 
+The configuration value for _hbase.masters_ is a comma separated list of _host:port_ values. If no port value is specified, the default of _16000_ is assumed.
+
+Usually this configuration is kept out in the _hbase-site.xml_ and is picked up by the client from the `CLASSPATH`.
+
+If you are configuring an IDE to run an HBase client, you should include the _conf/_ directory on your classpath so _hbase-site.xml_ settings can be found (or add _src/test/resources_ to pick up the hbase-site.xml used by tests).
+
+For Java applications using Maven, including the hbase-shaded-client module is the recommended dependency when connecting to a cluster:
+[source,xml]
 
 Review comment:
   ya seems out of place but the section is named "Client configuration and dependencies connecting to an HBase cluster".. so probably was intentional. 

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


With regards,
Apache Git Services