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