HBASE-26412 Handle sink failure in RegionReplicationSink (#3815)
[hbase.git] / hbase-server / src / main / java / org / apache / hadoop / hbase / regionserver / SplitRequest.java
blobd699caed7cd7cbb22d8138815176dfdaae51af89
1 /**
3 * Licensed to the Apache Software Foundation (ASF) under one
4 * or more contributor license agreements. See the NOTICE file
5 * distributed with this work for additional information
6 * regarding copyright ownership. The ASF licenses this file
7 * to you under the Apache License, Version 2.0 (the
8 * "License"); you may not use this file except in compliance
9 * with the License. You may obtain a copy of the License at
11 * http://www.apache.org/licenses/LICENSE-2.0
13 * Unless required by applicable law or agreed to in writing, software
14 * distributed under the License is distributed on an "AS IS" BASIS,
15 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
16 * See the License for the specific language governing permissions and
17 * limitations under the License.
19 package org.apache.hadoop.hbase.regionserver;
21 import java.security.PrivilegedAction;
22 import org.apache.hadoop.hbase.HConstants;
23 import org.apache.hadoop.hbase.TableName;
24 import org.apache.hadoop.hbase.client.RegionInfo;
25 import org.apache.hadoop.hbase.client.RegionInfoBuilder;
26 import org.apache.hadoop.hbase.regionserver.RegionServerServices.RegionStateTransitionContext;
27 import org.apache.hadoop.hbase.security.User;
28 import org.apache.hadoop.hbase.util.Bytes;
29 import org.apache.yetus.audience.InterfaceAudience;
30 import org.slf4j.Logger;
31 import org.slf4j.LoggerFactory;
33 import org.apache.hbase.thirdparty.com.google.common.base.Preconditions;
35 import org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionStateTransition.TransitionCode;
37 /**
38 * Handles processing region splits. Put in a queue, owned by HRegionServer.
40 @InterfaceAudience.Private
41 class SplitRequest implements Runnable {
42 private static final Logger LOG = LoggerFactory.getLogger(SplitRequest.class);
43 private final RegionInfo parent;
44 private final byte[] midKey;
45 private final HRegionServer server;
46 private final User user;
48 SplitRequest(Region region, byte[] midKey, HRegionServer hrs, User user) {
49 Preconditions.checkNotNull(hrs);
50 this.parent = region.getRegionInfo();
51 this.midKey = midKey;
52 this.server = hrs;
53 this.user = user;
56 @Override
57 public String toString() {
58 return "regionName=" + parent + ", midKey=" + Bytes.toStringBinary(midKey);
61 private void doSplitting() {
62 server.getMetrics().incrSplitRequest();
63 if (user != null && user.getUGI() != null) {
64 user.getUGI().doAs((PrivilegedAction<Void>) () -> {
65 requestRegionSplit();
66 return null;
67 });
68 } else {
69 requestRegionSplit();
73 private void requestRegionSplit() {
74 final TableName table = parent.getTable();
75 final RegionInfo hri_a = RegionInfoBuilder.newBuilder(table)
76 .setStartKey(parent.getStartKey())
77 .setEndKey(midKey)
78 .build();
79 final RegionInfo hri_b = RegionInfoBuilder.newBuilder(table)
80 .setStartKey(midKey)
81 .setEndKey(parent.getEndKey())
82 .build();
83 // Send the split request to the master. the master will do the validation on the split-key.
84 // The parent region will be unassigned and the two new regions will be assigned.
85 // hri_a and hri_b objects may not reflect the regions that will be created, those objects
86 // are created just to pass the information to the reportRegionStateTransition().
87 if (!server.reportRegionStateTransition(new RegionStateTransitionContext(
88 TransitionCode.READY_TO_SPLIT, HConstants.NO_SEQNUM, -1, parent, hri_a, hri_b))) {
89 LOG.error("Unable to ask master to split " + parent.getRegionNameAsString());
93 @Override
94 public void run() {
95 if (this.server.isStopping() || this.server.isStopped()) {
96 LOG.debug("Skipping split because server is stopping=" +
97 this.server.isStopping() + " or stopped=" + this.server.isStopped());
98 return;
101 doSplitting();