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.master.webapp;
019
020import java.io.UnsupportedEncodingException;
021import java.net.URLDecoder;
022import java.net.URLEncoder;
023import java.nio.charset.StandardCharsets;
024import java.util.ArrayList;
025import java.util.Collection;
026import java.util.Iterator;
027import java.util.LinkedList;
028import java.util.List;
029import java.util.stream.StreamSupport;
030import javax.servlet.http.HttpServletRequest;
031import org.apache.commons.lang3.StringUtils;
032import org.apache.commons.lang3.builder.ToStringBuilder;
033import org.apache.commons.lang3.builder.ToStringStyle;
034import org.apache.hadoop.hbase.CompareOperator;
035import org.apache.hadoop.hbase.HConstants;
036import org.apache.hadoop.hbase.TableName;
037import org.apache.hadoop.hbase.client.AdvancedScanResultConsumer;
038import org.apache.hadoop.hbase.client.AsyncConnection;
039import org.apache.hadoop.hbase.client.AsyncTable;
040import org.apache.hadoop.hbase.client.ResultScanner;
041import org.apache.hadoop.hbase.client.Scan;
042import org.apache.hadoop.hbase.filter.Filter;
043import org.apache.hadoop.hbase.filter.FilterList;
044import org.apache.hadoop.hbase.filter.PrefixFilter;
045import org.apache.hadoop.hbase.filter.SingleColumnValueFilter;
046import org.apache.hadoop.hbase.master.RegionState;
047import org.apache.hadoop.hbase.util.Bytes;
048import org.apache.yetus.audience.InterfaceAudience;
049import org.apache.hbase.thirdparty.com.google.common.collect.Iterators;
050import org.apache.hbase.thirdparty.io.netty.handler.codec.http.QueryStringEncoder;
051
052/**
053 * <p>
054 * Support class for the "Meta Entries" section in {@code resources/hbase-webapps/master/table.jsp}.
055 * </p>
056 * <p>
057 * <b>Interface</b>. This class's intended consumer is {@code table.jsp}. As such, it's primary
058 * interface is the active {@link HttpServletRequest}, from which it uses the {@code scan_*}
059 * request parameters. This class supports paging through an optionally filtered view of the
060 * contents of {@code hbase:meta}. Those filters and the pagination offset are specified via these
061 * request parameters. It provides helper methods for constructing pagination links.
062 * <ul>
063 *   <li>{@value #NAME_PARAM} - the name of the table requested. The only table of our concern here
064 *   is {@code hbase:meta}; any other value is effectively ignored by the giant conditional in the
065 *   jsp.</li>
066 *   <li>{@value #SCAN_LIMIT_PARAM} - specifies a limit on the number of region (replicas) rendered
067 *   on the by the table in a single request -- a limit on page size. This corresponds to the
068 *   number of {@link RegionReplicaInfo} objects produced by {@link Results#iterator()}. When a
069 *   value for {@code scan_limit} is invalid or not specified, the default value of
070 *   {@value #SCAN_LIMIT_DEFAULT} is used. In order to avoid excessive resource consumption, a
071 *   maximum value of {@value #SCAN_LIMIT_MAX} is enforced.</li>
072 *   <li>{@value #SCAN_REGION_STATE_PARAM} - an optional filter on {@link RegionState}.</li>
073 *   <li>{@value #SCAN_START_PARAM} - specifies the rowkey at which a scan should start. For usage
074 *   details, see the below section on <b>Pagination</b>.</li>
075 *   <li>{@value #SCAN_TABLE_PARAM} - specifies a filter on the values returned, limiting them to
076 *   regions from a specified table. This parameter is implemented as a prefix filter on the
077 *   {@link Scan}, so in effect it can be used for simple namespace and multi-table matches.</li>
078 * </ul>
079 * </p>
080 * <p>
081 * <b>Pagination</b>. A single page of results are made available via {@link #getResults()} / an
082 * instance of {@link Results}. Callers use its {@link Iterator} consume the page of
083 * {@link RegionReplicaInfo} instances, each of which represents a region or region replica. Helper
084 * methods are provided for building page navigation controls preserving the user's selected filter
085 * set: {@link #buildFirstPageUrl()}, {@link #buildNextPageUrl(byte[])}. Pagination is implemented
086 * using a simple offset + limit system. Offset is provided by the {@value #SCAN_START_PARAM},
087 * limit via {@value #SCAN_LIMIT_PARAM}. Under the hood, the {@link Scan} is constructed with
088 * {@link Scan#setMaxResultSize(long)} set to ({@value SCAN_LIMIT_PARAM} +1), while the
089 * {@link Results} {@link Iterator} honors {@value #SCAN_LIMIT_PARAM}. The +1 allows the caller to
090 * know if a "next page" is available via {@link Results#hasMoreResults()}. Note that this
091 * pagination strategy is incomplete when it comes to region replicas and can potentially omit
092 * rendering replicas that fall between the last rowkey offset and {@code replicaCount % page size}.
093 * </p>
094 * <p>
095 * <b>Error Messages</b>. Any time there's an error parsing user input, a message will be populated
096 * in {@link #getErrorMessages()}. Any fields which produce an error will have their filter values
097 * set to the default, except for a value of {@value  #SCAN_LIMIT_PARAM} that exceeds
098 * {@value #SCAN_LIMIT_MAX}, in which case {@value #SCAN_LIMIT_MAX} is used.
099 * </p>
100 */
101@InterfaceAudience.Private
102public class MetaBrowser {
103  public static final String NAME_PARAM = "name";
104  public static final String SCAN_LIMIT_PARAM = "scan_limit";
105  public static final String SCAN_REGION_STATE_PARAM = "scan_region_state";
106  public static final String SCAN_START_PARAM = "scan_start";
107  public static final String SCAN_TABLE_PARAM = "scan_table";
108
109  public static final int SCAN_LIMIT_DEFAULT = 10;
110  public static final int SCAN_LIMIT_MAX = 10_000;
111
112  private final AsyncConnection connection;
113  private final HttpServletRequest request;
114  private final List<String> errorMessages;
115  private final String name;
116  private final Integer scanLimit;
117  private final RegionState.State scanRegionState;
118  private final byte[] scanStart;
119  private final TableName scanTable;
120
121  public MetaBrowser(final AsyncConnection connection, final HttpServletRequest request) {
122    this.connection = connection;
123    this.request = request;
124    this.errorMessages = new LinkedList<>();
125    this.name = resolveName(request);
126    this.scanLimit = resolveScanLimit(request);
127    this.scanRegionState = resolveScanRegionState(request);
128    this.scanStart = resolveScanStart(request);
129    this.scanTable = resolveScanTable(request);
130  }
131
132  public List<String> getErrorMessages() {
133    return errorMessages;
134  }
135
136  public String getName() {
137    return name;
138  }
139
140  public Integer getScanLimit() {
141    return scanLimit;
142  }
143
144  public byte[] getScanStart() {
145    return scanStart;
146  }
147
148  public RegionState.State getScanRegionState() {
149    return scanRegionState;
150  }
151
152  public TableName getScanTable() {
153    return scanTable;
154  }
155
156  public Results getResults() {
157    final AsyncTable<AdvancedScanResultConsumer> asyncTable =
158      connection.getTable(TableName.META_TABLE_NAME);
159    return new Results(asyncTable.getScanner(buildScan()));
160  }
161
162  @Override
163  public String toString() {
164    return new ToStringBuilder(this, ToStringStyle.SHORT_PREFIX_STYLE)
165      .append("scanStart", scanStart)
166      .append("scanLimit", scanLimit)
167      .append("scanTable", scanTable)
168      .append("scanRegionState", scanRegionState)
169      .toString();
170  }
171
172  private static String resolveName(final HttpServletRequest request) {
173    return resolveRequestParameter(request, NAME_PARAM);
174  }
175
176  private Integer resolveScanLimit(final HttpServletRequest request) {
177    final String requestValueStr = resolveRequestParameter(request, SCAN_LIMIT_PARAM);
178    if (StringUtils.isBlank(requestValueStr)) {
179      return null;
180    }
181
182    final Integer requestValue = tryParseInt(requestValueStr);
183    if (requestValue == null) {
184      errorMessages.add(buildScanLimitMalformedErrorMessage(requestValueStr));
185      return null;
186    }
187    if (requestValue <= 0) {
188      errorMessages.add(buildScanLimitLTEQZero(requestValue));
189      return SCAN_LIMIT_DEFAULT;
190    }
191
192    final int truncatedValue = Math.min(requestValue, SCAN_LIMIT_MAX);
193    if (requestValue != truncatedValue) {
194      errorMessages.add(buildScanLimitExceededErrorMessage(requestValue));
195    }
196    return truncatedValue;
197  }
198
199  private RegionState.State resolveScanRegionState(final HttpServletRequest request) {
200    final String requestValueStr = resolveRequestParameter(request, SCAN_REGION_STATE_PARAM);
201    if (requestValueStr == null) {
202      return null;
203    }
204    final RegionState.State requestValue = tryValueOf(RegionState.State.class, requestValueStr);
205    if (requestValue == null) {
206      errorMessages.add(buildScanRegionStateMalformedErrorMessage(requestValueStr));
207      return null;
208    }
209    return requestValue;
210  }
211
212  private static byte[] resolveScanStart(final HttpServletRequest request) {
213    // TODO: handle replicas that fall between the last rowkey and pagination limit.
214    final String requestValue = resolveRequestParameter(request, SCAN_START_PARAM);
215    if (requestValue == null) {
216      return null;
217    }
218    return Bytes.toBytesBinary(requestValue);
219  }
220
221  private static TableName resolveScanTable(final HttpServletRequest request) {
222    final String requestValue = resolveRequestParameter(request, SCAN_TABLE_PARAM);
223    if (requestValue == null) {
224      return null;
225    }
226    return TableName.valueOf(requestValue);
227  }
228
229  private static String resolveRequestParameter(final HttpServletRequest request,
230    final String param) {
231    if (request == null) {
232      return null;
233    }
234    final String requestValueStrEnc = request.getParameter(param);
235    if (StringUtils.isBlank(requestValueStrEnc)) {
236      return null;
237    }
238    return urlDecode(requestValueStrEnc);
239  }
240
241  private static Filter buildTableFilter(final TableName tableName) {
242    return new PrefixFilter(tableName.toBytes());
243  }
244
245  private static Filter buildScanRegionStateFilter(final RegionState.State state) {
246    return new SingleColumnValueFilter(
247      HConstants.CATALOG_FAMILY,
248      HConstants.STATE_QUALIFIER,
249      CompareOperator.EQUAL,
250      // use the same serialization strategy as found in MetaTableAccessor#addRegionStateToPut
251      Bytes.toBytes(state.name()));
252  }
253
254  private Filter buildScanFilter() {
255    if (scanTable == null && scanRegionState == null) {
256      return null;
257    }
258
259    final List<Filter> filters = new ArrayList<>(2);
260    if (scanTable != null) {
261      filters.add(buildTableFilter(scanTable));
262    }
263    if (scanRegionState != null) {
264      filters.add(buildScanRegionStateFilter(scanRegionState));
265    }
266    if (filters.size() == 1) {
267      return filters.get(0);
268    }
269    return new FilterList(FilterList.Operator.MUST_PASS_ALL, filters);
270  }
271
272  private Scan buildScan() {
273    final Scan metaScan = new Scan()
274      .addFamily(HConstants.CATALOG_FAMILY)
275      .readVersions(1)
276      .setLimit((scanLimit != null ? scanLimit : SCAN_LIMIT_DEFAULT) + 1);
277    if (scanStart != null) {
278      metaScan.withStartRow(scanStart, false);
279    }
280    final Filter filter = buildScanFilter();
281    if (filter != null) {
282      metaScan.setFilter(filter);
283    }
284    return metaScan;
285  }
286
287  /**
288   * Adds {@code value} to {@code encoder} under {@code paramName} when {@code value} is non-null.
289   */
290  private void addParam(final QueryStringEncoder encoder, final String paramName,
291    final Object value) {
292    if (value != null) {
293      encoder.addParam(paramName, value.toString());
294    }
295  }
296
297  private QueryStringEncoder buildFirstPageEncoder() {
298    final QueryStringEncoder encoder =
299      new QueryStringEncoder(request.getRequestURI());
300    addParam(encoder, NAME_PARAM, name);
301    addParam(encoder, SCAN_LIMIT_PARAM, scanLimit);
302    addParam(encoder, SCAN_REGION_STATE_PARAM, scanRegionState);
303    addParam(encoder, SCAN_TABLE_PARAM, scanTable);
304    return encoder;
305  }
306
307  public String buildFirstPageUrl() {
308    return buildFirstPageEncoder().toString();
309  }
310
311  static String buildStartParamFrom(final byte[] lastRow) {
312    if (lastRow == null) {
313      return null;
314    }
315    return urlEncode(Bytes.toStringBinary(lastRow));
316  }
317
318  public String buildNextPageUrl(final byte[] lastRow) {
319    final QueryStringEncoder encoder = buildFirstPageEncoder();
320    final String startRow = buildStartParamFrom(lastRow);
321    addParam(encoder, SCAN_START_PARAM, startRow);
322    return encoder.toString();
323  }
324
325  private static String urlEncode(final String val) {
326    if (StringUtils.isEmpty(val)) {
327      return null;
328    }
329    try {
330      return URLEncoder.encode(val, StandardCharsets.UTF_8.toString());
331    } catch (UnsupportedEncodingException e) {
332      return null;
333    }
334  }
335
336  private static String urlDecode(final String val) {
337    if (StringUtils.isEmpty(val)) {
338      return null;
339    }
340    try {
341      return URLDecoder.decode(val, StandardCharsets.UTF_8.toString());
342    } catch (UnsupportedEncodingException e) {
343      return null;
344    }
345  }
346
347  private static Integer tryParseInt(final String val) {
348    if (StringUtils.isEmpty(val)) {
349      return null;
350    }
351    try {
352      return Integer.parseInt(val);
353    } catch (NumberFormatException e) {
354      return null;
355    }
356  }
357
358  private static <T extends Enum<T>> T tryValueOf(final Class<T> clazz,
359    final String value) {
360    if (clazz == null || value == null) {
361      return null;
362    }
363    try {
364      return Enum.valueOf(clazz, value);
365    } catch (IllegalArgumentException e) {
366      return null;
367    }
368  }
369
370  private static String buildScanLimitExceededErrorMessage(final int requestValue) {
371    return String.format(
372      "Requested SCAN_LIMIT value %d exceeds maximum value %d.", requestValue, SCAN_LIMIT_MAX);
373  }
374
375  private static String buildScanLimitMalformedErrorMessage(final String requestValue) {
376    return String.format(
377      "Requested SCAN_LIMIT value '%s' cannot be parsed as an integer.", requestValue);
378  }
379
380  private static String buildScanLimitLTEQZero(final int requestValue) {
381    return String.format("Requested SCAN_LIMIT value %d is <= 0.", requestValue);
382  }
383
384  private static String buildScanRegionStateMalformedErrorMessage(final String requestValue) {
385    return String.format(
386      "Requested SCAN_REGION_STATE value '%s' cannot be parsed as a RegionState.", requestValue);
387  }
388
389  /**
390   * Encapsulates the results produced by this {@link MetaBrowser} instance.
391   */
392  public final class Results implements AutoCloseable, Iterable<RegionReplicaInfo> {
393
394    private final ResultScanner resultScanner;
395    private final Iterator<RegionReplicaInfo> sourceIterator;
396
397    private Results(final ResultScanner resultScanner) {
398      this.resultScanner = resultScanner;
399      this.sourceIterator =  StreamSupport.stream(resultScanner.spliterator(), false)
400        .map(RegionReplicaInfo::from)
401        .flatMap(Collection::stream)
402        .iterator();
403    }
404
405    /**
406     * @return {@code true} when the underlying {@link ResultScanner} is not yet exhausted,
407     *   {@code false} otherwise.
408     */
409    public boolean hasMoreResults() {
410      return sourceIterator.hasNext();
411    }
412
413    @Override
414    public void close() {
415      if (resultScanner != null) {
416        resultScanner.close();
417      }
418    }
419
420    @Override public Iterator<RegionReplicaInfo> iterator() {
421      return Iterators.limit(sourceIterator, scanLimit != null ? scanLimit : SCAN_LIMIT_DEFAULT);
422    }
423  }
424}