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.util;
019
020import java.io.IOException;
021import org.apache.hadoop.conf.Configuration;
022import org.apache.hadoop.fs.FileSystem;
023import org.apache.hadoop.fs.Path;
024import org.apache.hadoop.hbase.TableName;
025import org.apache.hadoop.hbase.client.Admin;
026import org.apache.hadoop.hbase.client.RegionInfo;
027import org.apache.hadoop.hbase.client.RegionReplicaUtil;
028import org.apache.hadoop.hbase.io.HFileLink;
029import org.apache.hadoop.hbase.io.Reference;
030import org.apache.hadoop.hbase.master.MasterServices;
031import org.apache.hadoop.hbase.regionserver.HRegion;
032import org.apache.hadoop.hbase.regionserver.StoreFileInfo;
033import org.apache.hadoop.hbase.replication.ReplicationException;
034import org.apache.hadoop.hbase.replication.ReplicationPeerConfig;
035import org.apache.hadoop.hbase.replication.regionserver.RegionReplicaReplicationEndpoint;
036import org.apache.hadoop.hbase.zookeeper.ZKConfig;
037import org.apache.yetus.audience.InterfaceAudience;
038import org.slf4j.Logger;
039import org.slf4j.LoggerFactory;
040
041/**
042 * Similar to {@link RegionReplicaUtil} but for the server side
043 */
044@InterfaceAudience.Private
045public class ServerRegionReplicaUtil extends RegionReplicaUtil {
046
047  private static final Logger LOG = LoggerFactory.getLogger(ServerRegionReplicaUtil.class);
048
049  /**
050   * Whether asynchronous WAL replication to the secondary region replicas is enabled or not. If
051   * this is enabled, a replication peer named "region_replica_replication" will be created which
052   * will tail the logs and replicate the mutatations to region replicas for tables that have region
053   * replication > 1. If this is enabled once, disabling this replication also requires disabling
054   * the replication peer using shell or {@link Admin} java class. Replication to secondary region
055   * replicas works over standard inter-cluster replication.ยท
056   */
057  public static final String REGION_REPLICA_REPLICATION_CONF_KEY =
058    "hbase.region.replica.replication.enabled";
059  private static final boolean DEFAULT_REGION_REPLICA_REPLICATION = false;
060  public static final String REGION_REPLICA_REPLICATION_PEER = "region_replica_replication";
061
062  /**
063   * Same as for {@link #REGION_REPLICA_REPLICATION_CONF_KEY} but for catalog replication.
064   */
065  public static final String REGION_REPLICA_REPLICATION_CATALOG_CONF_KEY =
066    "hbase.region.replica.replication.catalog.enabled";
067  private static final boolean DEFAULT_REGION_REPLICA_REPLICATION_CATALOG = false;
068
069  /**
070   * Enables or disables refreshing store files of secondary region replicas when the memory is
071   * above the global memstore lower limit. Refreshing the store files means that we will do a file
072   * list of the primary regions store files, and pick up new files. Also depending on the store
073   * files, we can drop some memstore contents which will free up memory.
074   */
075  public static final String REGION_REPLICA_STORE_FILE_REFRESH =
076    "hbase.region.replica.storefile.refresh";
077  private static final boolean DEFAULT_REGION_REPLICA_STORE_FILE_REFRESH = true;
078
079  /**
080   * The multiplier to use when we want to refresh a secondary region instead of flushing a primary
081   * region. Default value assumes that for doing the file refresh, the biggest secondary should be
082   * 4 times bigger than the biggest primary.
083   */
084  public static final String REGION_REPLICA_STORE_FILE_REFRESH_MEMSTORE_MULTIPLIER =
085    "hbase.region.replica.storefile.refresh.memstore.multiplier";
086  private static final double DEFAULT_REGION_REPLICA_STORE_FILE_REFRESH_MEMSTORE_MULTIPLIER = 4;
087
088  /**
089   * Returns the regionInfo object to use for interacting with the file system.
090   * @return An RegionInfo object to interact with the filesystem
091   */
092  public static RegionInfo getRegionInfoForFs(RegionInfo regionInfo) {
093    if (regionInfo == null) {
094      return null;
095    }
096    return RegionReplicaUtil.getRegionInfoForDefaultReplica(regionInfo);
097  }
098
099  /**
100   * Returns whether this region replica can accept writes.
101   * @param region the HRegion object
102   * @return whether the replica is read only
103   */
104  public static boolean isReadOnly(HRegion region) {
105    return region.getTableDescriptor().isReadOnly() || !isDefaultReplica(region.getRegionInfo());
106  }
107
108  /**
109   * Returns whether to replay the recovered edits to flush the results. Currently secondary region
110   * replicas do not replay the edits, since it would cause flushes which might affect the primary
111   * region. Primary regions even opened in read only mode should replay the edits.
112   * @param region the HRegion object
113   * @return whether recovered edits should be replayed.
114   */
115  public static boolean shouldReplayRecoveredEdits(HRegion region) {
116    return isDefaultReplica(region.getRegionInfo());
117  }
118
119  /**
120   * Returns a StoreFileInfo from the given FileStatus. Secondary replicas refer to the files of the
121   * primary region, so an HFileLink is used to construct the StoreFileInfo. This way ensures that
122   * the secondary will be able to continue reading the store files even if they are moved to
123   * archive after compaction
124   */
125  public static StoreFileInfo getStoreFileInfo(Configuration conf, FileSystem fs,
126    RegionInfo regionInfo, RegionInfo regionInfoForFs, String familyName, Path path)
127    throws IOException {
128
129    // if this is a primary region, just return the StoreFileInfo constructed from path
130    if (RegionInfo.COMPARATOR.compare(regionInfo, regionInfoForFs) == 0) {
131      return new StoreFileInfo(conf, fs, path, true);
132    }
133
134    // else create a store file link. The link file does not exists on filesystem though.
135    if (HFileLink.isHFileLink(path) || StoreFileInfo.isHFile(path)) {
136      HFileLink link = HFileLink.build(conf, regionInfoForFs.getTable(),
137        regionInfoForFs.getEncodedName(), familyName, path.getName());
138      return new StoreFileInfo(conf, fs, link.getFileStatus(fs), link);
139    } else if (StoreFileInfo.isReference(path)) {
140      Reference reference = Reference.read(fs, path);
141      Path referencePath = StoreFileInfo.getReferredToFile(path);
142      if (HFileLink.isHFileLink(referencePath)) {
143        // HFileLink Reference
144        HFileLink link = HFileLink.buildFromHFileLinkPattern(conf, referencePath);
145        return new StoreFileInfo(conf, fs, link.getFileStatus(fs), reference, link);
146      } else {
147        // Reference
148        HFileLink link = HFileLink.build(conf, regionInfoForFs.getTable(),
149          regionInfoForFs.getEncodedName(), familyName, path.getName());
150        return new StoreFileInfo(conf, fs, link.getFileStatus(fs), reference);
151      }
152    } else {
153      throw new IOException("path=" + path + " doesn't look like a valid StoreFile");
154    }
155  }
156
157  /**
158   * Create replication peer for replicating user-space Region Read Replicas. This methods should
159   * only be called at master side.
160   */
161  public static void setupRegionReplicaReplication(MasterServices services)
162    throws IOException, ReplicationException {
163    if (!isRegionReplicaReplicationEnabled(services.getConfiguration())) {
164      return;
165    }
166    if (
167      services.getReplicationPeerManager().getPeerConfig(REGION_REPLICA_REPLICATION_PEER)
168        .isPresent()
169    ) {
170      return;
171    }
172    LOG.info("Region replica replication peer id=" + REGION_REPLICA_REPLICATION_PEER
173      + " not exist. Creating...");
174    ReplicationPeerConfig peerConfig = ReplicationPeerConfig.newBuilder()
175      .setClusterKey(ZKConfig.getZooKeeperClusterKey(services.getConfiguration()))
176      .setReplicationEndpointImpl(RegionReplicaReplicationEndpoint.class.getName()).build();
177    services.addReplicationPeer(REGION_REPLICA_REPLICATION_PEER, peerConfig, true);
178  }
179
180  /**
181   * @return True if Region Read Replica is enabled for <code>tn</code> (whether hbase:meta or
182   *         user-space tables).
183   */
184  public static boolean isRegionReplicaReplicationEnabled(Configuration conf, TableName tn) {
185    return isMetaRegionReplicaReplicationEnabled(conf, tn)
186      || isRegionReplicaReplicationEnabled(conf);
187  }
188
189  /** Returns True if Region Read Replica is enabled for user-space tables. */
190  private static boolean isRegionReplicaReplicationEnabled(Configuration conf) {
191    return conf.getBoolean(REGION_REPLICA_REPLICATION_CONF_KEY, DEFAULT_REGION_REPLICA_REPLICATION);
192  }
193
194  /** Returns True if hbase:meta Region Read Replica is enabled. */
195  public static boolean isMetaRegionReplicaReplicationEnabled(Configuration conf, TableName tn) {
196    return TableName.isMetaTableName(tn) && conf.getBoolean(
197      REGION_REPLICA_REPLICATION_CATALOG_CONF_KEY, DEFAULT_REGION_REPLICA_REPLICATION_CATALOG);
198  }
199
200  /** Returns True if wait for primary to flush is enabled for user-space tables. */
201  public static boolean isRegionReplicaWaitForPrimaryFlushEnabled(Configuration conf) {
202    return conf.getBoolean(REGION_REPLICA_WAIT_FOR_PRIMARY_FLUSH_CONF_KEY,
203      DEFAULT_REGION_REPLICA_WAIT_FOR_PRIMARY_FLUSH);
204  }
205
206  /** Returns True if we are to refresh user-space hfiles in Region Read Replicas. */
207  public static boolean isRegionReplicaStoreFileRefreshEnabled(Configuration conf) {
208    return conf.getBoolean(REGION_REPLICA_STORE_FILE_REFRESH,
209      DEFAULT_REGION_REPLICA_STORE_FILE_REFRESH);
210  }
211
212  public static double getRegionReplicaStoreFileRefreshMultiplier(Configuration conf) {
213    return conf.getDouble(REGION_REPLICA_STORE_FILE_REFRESH_MEMSTORE_MULTIPLIER,
214      DEFAULT_REGION_REPLICA_STORE_FILE_REFRESH_MEMSTORE_MULTIPLIER);
215  }
216
217}