001/*
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hbase.client;
019
020import java.io.IOException;
021import java.util.List;
022import org.apache.hadoop.conf.Configuration;
023import org.apache.hadoop.hbase.HRegionLocation;
024import org.apache.hadoop.hbase.MasterNotRunningException;
025import org.apache.hadoop.hbase.RegionLocations;
026import org.apache.hadoop.hbase.ServerName;
027import org.apache.hadoop.hbase.TableName;
028import org.apache.hadoop.hbase.ZooKeeperConnectionException;
029import org.apache.hadoop.hbase.client.backoff.ClientBackoffPolicy;
030import org.apache.hadoop.hbase.ipc.RpcControllerFactory;
031import org.apache.hadoop.hbase.security.User;
032import org.apache.yetus.audience.InterfaceAudience;
033
034import org.apache.hadoop.hbase.shaded.protobuf.generated.AdminProtos.AdminService;
035import org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.ClientService;
036
037/** Internal methods on Connection that should not be used by user code. */
038@InterfaceAudience.Private
039// NOTE: Although this class is public, this class is meant to be used directly from internal
040// classes and unit tests only.
041public interface ClusterConnection extends Connection {
042
043  /**
044   * Key for configuration in Configuration whose value is the class we implement making a new
045   * Connection instance.
046   */
047  String HBASE_CLIENT_CONNECTION_IMPL = "hbase.client.connection.impl";
048
049  /**
050   * @return - true if the master server is running
051   * @deprecated this has been deprecated without a replacement
052   */
053  @Deprecated
054  boolean isMasterRunning() throws MasterNotRunningException, ZooKeeperConnectionException;
055
056  /**
057   * Use this api to check if the table has been created with the specified number of splitkeys
058   * which was used while creating the given table. Note : If this api is used after a table's
059   * region gets splitted, the api may return false. n * tableName n * splitKeys used while creating
060   * table n * if a remote or network exception occurs
061   */
062  boolean isTableAvailable(TableName tableName, byte[][] splitKeys) throws IOException;
063
064  /**
065   * A table that isTableEnabled == false and isTableDisabled == false is possible. This happens
066   * when a table has a lot of regions that must be processed.
067   * @param tableName table name
068   * @return true if the table is enabled, false otherwise
069   * @throws IOException if a remote or network exception occurs
070   */
071  boolean isTableEnabled(TableName tableName) throws IOException;
072
073  /**
074   * Check if a table is disabled.
075   * @param tableName table name
076   * @return true if the table is disabled, false otherwise
077   * @throws IOException if a remote or network exception occurs
078   */
079  boolean isTableDisabled(TableName tableName) throws IOException;
080
081  /**
082   * Retrieve TableState, represent current table state.
083   * @param tableName table state for
084   * @return state of the table
085   */
086  TableState getTableState(TableName tableName) throws IOException;
087
088  /**
089   * Find the location of the region of <i>tableName</i> that <i>row</i> lives in.
090   * @param tableName name of the table <i>row</i> is in
091   * @param row       row key you're trying to find the region of
092   * @return HRegionLocation that describes where to find the region in question
093   * @throws IOException if a remote or network exception occurs
094   */
095  HRegionLocation locateRegion(final TableName tableName, final byte[] row) throws IOException;
096
097  /**
098   * Clear the region location cache.
099   * @deprecated {@link #clearRegionLocationCache()} instead.
100   */
101  @Deprecated
102  default void clearRegionCache() {
103    clearRegionLocationCache();
104  }
105
106  void cacheLocation(final TableName tableName, final RegionLocations location);
107
108  /**
109   * Allows flushing the region cache of all locations that pertain to <code>tableName</code>
110   * @param tableName Name of the table whose regions we are to remove from cache.
111   */
112  void clearRegionCache(final TableName tableName);
113
114  /**
115   * Deletes cached locations for the specific region.
116   * @param location The location object for the region, to be purged from cache.
117   */
118  void deleteCachedRegionLocation(final HRegionLocation location);
119
120  /**
121   * Find the location of the region of <i>tableName</i> that <i>row</i> lives in, ignoring any
122   * value that might be in the cache.
123   * @param tableName name of the table <i>row</i> is in
124   * @param row       row key you're trying to find the region of
125   * @return HRegionLocation that describes where to find the region in question
126   * @throws IOException if a remote or network exception occurs
127   */
128  HRegionLocation relocateRegion(final TableName tableName, final byte[] row) throws IOException;
129
130  /**
131   * Find the location of the region of <i>tableName</i> that <i>row</i> lives in, ignoring any
132   * value that might be in the cache.
133   * @param tableName name of the table <i>row</i> is in
134   * @param row       row key you're trying to find the region of
135   * @param replicaId the replicaId of the region
136   * @return RegionLocations that describe where to find the region in question
137   * @throws IOException if a remote or network exception occurs
138   */
139  RegionLocations relocateRegion(final TableName tableName, final byte[] row, int replicaId)
140    throws IOException;
141
142  /**
143   * Update the location cache. This is used internally by HBase, in most cases it should not be
144   * used by the client application.
145   * @param tableName  the table name
146   * @param regionName the region name
147   * @param rowkey     the row
148   * @param exception  the exception if any. Can be null.
149   * @param source     the previous location
150   */
151  void updateCachedLocations(TableName tableName, byte[] regionName, byte[] rowkey,
152    Object exception, ServerName source);
153
154  /**
155   * Gets the location of the region of <i>regionName</i>.
156   * @param regionName name of the region to locate
157   * @return HRegionLocation that describes where to find the region in question
158   * @throws IOException if a remote or network exception occurs
159   */
160  HRegionLocation locateRegion(final byte[] regionName) throws IOException;
161
162  /**
163   * Gets the locations of all regions in the specified table, <i>tableName</i>.
164   * @param tableName table to get regions of
165   * @return list of region locations for all regions of table
166   * @throws IOException if IO failure occurs
167   */
168  List<HRegionLocation> locateRegions(final TableName tableName) throws IOException;
169
170  /**
171   * Gets the locations of all regions in the specified table, <i>tableName</i>.
172   * @param tableName table to get regions of
173   * @param useCache  Should we use the cache to retrieve the region information.
174   * @param offlined  True if we are to include offlined regions, false and we'll leave out offlined
175   *                  regions from returned list.
176   * @return list of region locations for all regions of table
177   * @throws IOException if IO failure occurs
178   */
179  List<HRegionLocation> locateRegions(final TableName tableName, final boolean useCache,
180    final boolean offlined) throws IOException;
181
182  /**
183   * Gets the locations of the region in the specified table, <i>tableName</i>, for a given row.
184   * @param tableName table to get regions of
185   * @param row       the row
186   * @param useCache  Should we use the cache to retrieve the region information.
187   * @param retry     do we retry
188   * @return region locations for this row.
189   * @throws IOException if IO failure occurs
190   */
191  RegionLocations locateRegion(TableName tableName, byte[] row, boolean useCache, boolean retry)
192    throws IOException;
193
194  /**
195   * Gets the locations of the region in the specified table, <i>tableName</i>, for a given row.
196   * @param tableName table to get regions of
197   * @param row       the row
198   * @param useCache  Should we use the cache to retrieve the region information.
199   * @param retry     do we retry
200   * @param replicaId the replicaId for the region
201   * @return region locations for this row.
202   * @throws IOException if IO failure occurs
203   */
204  RegionLocations locateRegion(TableName tableName, byte[] row, boolean useCache, boolean retry,
205    int replicaId) throws IOException;
206
207  /**
208   * Returns a {@link MasterKeepAliveConnection} to the active master
209   */
210  MasterKeepAliveConnection getMaster() throws IOException;
211
212  /**
213   * Get the admin service for master.
214   */
215  AdminService.BlockingInterface getAdminForMaster() throws IOException;
216
217  /**
218   * Establishes a connection to the region server at the specified address.
219   * @param serverName the region server to connect to
220   * @return proxy for HRegionServer
221   * @throws IOException if a remote or network exception occurs
222   */
223  AdminService.BlockingInterface getAdmin(final ServerName serverName) throws IOException;
224
225  /**
226   * Establishes a connection to the region server at the specified address, and returns a region
227   * client protocol.
228   * @param serverName the region server to connect to
229   * @return ClientProtocol proxy for RegionServer
230   * @throws IOException if a remote or network exception occurs
231   */
232  ClientService.BlockingInterface getClient(final ServerName serverName) throws IOException;
233
234  /**
235   * Find region location hosting passed row
236   * @param tableName table name
237   * @param row       Row to find.
238   * @param reload    If true do not use cache, otherwise bypass.
239   * @return Location of row.
240   * @throws IOException if a remote or network exception occurs
241   */
242  HRegionLocation getRegionLocation(TableName tableName, byte[] row, boolean reload)
243    throws IOException;
244
245  /**
246   * Clear any caches that pertain to server name <code>sn</code>.
247   * @param sn A server name
248   */
249  void clearCaches(final ServerName sn);
250
251  /**
252   * Returns Nonce generator for this ClusterConnection; may be null if disabled in configuration.
253   */
254  NonceGenerator getNonceGenerator();
255
256  /** Returns Default AsyncProcess associated with this connection. */
257  AsyncProcess getAsyncProcess();
258
259  /**
260   * Returns a new RpcRetryingCallerFactory from the given {@link Configuration}. This
261   * RpcRetryingCallerFactory lets the users create {@link RpcRetryingCaller}s which can be
262   * intercepted with the configured {@link RetryingCallerInterceptor}
263   * @param conf configuration n
264   */
265  RpcRetryingCallerFactory getNewRpcRetryingCallerFactory(Configuration conf);
266
267  /** Returns Connection's RpcRetryingCallerFactory instance */
268  RpcRetryingCallerFactory getRpcRetryingCallerFactory();
269
270  /** Returns Connection's RpcControllerFactory instance */
271  RpcControllerFactory getRpcControllerFactory();
272
273  /** Returns a ConnectionConfiguration object holding parsed configuration values */
274  ConnectionConfiguration getConnectionConfiguration();
275
276  /** Returns the current statistics tracker associated with this connection */
277  ServerStatisticTracker getStatisticsTracker();
278
279  /** Returns the configured client backoff policy */
280  ClientBackoffPolicy getBackoffPolicy();
281
282  /** Returns the MetricsConnection instance associated with this connection. */
283  MetricsConnection getConnectionMetrics();
284
285  /**
286   * Returns true when this connection uses a {@link org.apache.hadoop.hbase.codec.Codec} and so
287   * supports cell blocks.
288   */
289  boolean hasCellBlockSupport();
290
291  /**
292   * Get the {@link User} associated with this connection. May be {@code null}.
293   */
294  User getUser();
295
296  /**
297   * Get the {@link ConnectionRegistry} used to orient this cluster.
298   */
299  ConnectionRegistry getConnectionRegistry();
300}