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.regionserver;
019
020import java.io.FileNotFoundException;
021import java.io.IOException;
022import java.io.InterruptedIOException;
023import java.net.ConnectException;
024import java.net.SocketTimeoutException;
025import java.util.Optional;
026import org.apache.commons.io.IOUtils;
027import org.apache.hadoop.conf.Configuration;
028import org.apache.hadoop.fs.FSDataInputStream;
029import org.apache.hadoop.fs.FSDataOutputStream;
030import org.apache.hadoop.fs.FileSystem;
031import org.apache.hadoop.fs.Path;
032import org.apache.hadoop.hbase.NotServingRegionException;
033import org.apache.hadoop.hbase.Server;
034import org.apache.hadoop.hbase.client.RetriesExhaustedException;
035import org.apache.hadoop.hbase.coordination.SplitLogWorkerCoordination;
036import org.apache.hadoop.hbase.regionserver.SplitLogWorker.TaskExecutor.Status;
037import org.apache.hadoop.hbase.replication.ReplicationPeerImpl;
038import org.apache.hadoop.hbase.replication.ReplicationUtils;
039import org.apache.hadoop.hbase.replication.SyncReplicationState;
040import org.apache.hadoop.hbase.util.CancelableProgressable;
041import org.apache.hadoop.hbase.util.CommonFSUtils;
042import org.apache.hadoop.hbase.util.ExceptionUtil;
043import org.apache.hadoop.hbase.util.FSUtils;
044import org.apache.hadoop.hbase.util.Pair;
045import org.apache.hadoop.hbase.wal.SyncReplicationWALProvider;
046import org.apache.hadoop.hbase.wal.WALFactory;
047import org.apache.hadoop.hbase.wal.WALSplitter;
048import org.apache.yetus.audience.InterfaceAudience;
049import org.slf4j.Logger;
050import org.slf4j.LoggerFactory;
051
052/**
053 * This worker is spawned in every regionserver, including master. The Worker waits for log
054 * splitting tasks to be put up by the {@link org.apache.hadoop.hbase.master.SplitLogManager}
055 * running in the master and races with other workers in other serves to acquire those tasks.
056 * The coordination is done via coordination engine.
057 * <p>
058 * If a worker has successfully moved the task from state UNASSIGNED to OWNED then it owns the task.
059 * It keeps heart beating the manager by periodically moving the task from UNASSIGNED to OWNED
060 * state. On success it moves the task to TASK_DONE. On unrecoverable error it moves task state to
061 * ERR. If it cannot continue but wants the master to retry the task then it moves the task state to
062 * RESIGNED.
063 * <p>
064 * The manager can take a task away from a worker by moving the task from OWNED to UNASSIGNED. In
065 * the absence of a global lock there is a unavoidable race here - a worker might have just finished
066 * its task when it is stripped of its ownership. Here we rely on the idempotency of the log
067 * splitting task for correctness
068 * @deprecated since 2.4.0 and in 3.0.0, to be removed in 4.0.0, replaced by procedure-based
069 *   distributed WAL splitter, see SplitWALRemoteProcedure
070 */
071@Deprecated
072@InterfaceAudience.Private
073public class SplitLogWorker implements Runnable {
074
075  private static final Logger LOG = LoggerFactory.getLogger(SplitLogWorker.class);
076
077  Thread worker;
078  // thread pool which executes recovery work
079  private final SplitLogWorkerCoordination coordination;
080  private final RegionServerServices server;
081
082  public SplitLogWorker(Server hserver, Configuration conf, RegionServerServices server,
083      TaskExecutor splitTaskExecutor) {
084    this.server = server;
085    this.coordination = hserver.getCoordinatedStateManager().getSplitLogWorkerCoordination();
086    coordination.init(server, conf, splitTaskExecutor, this);
087  }
088
089  public SplitLogWorker(Configuration conf, RegionServerServices server,
090      LastSequenceId sequenceIdChecker, WALFactory factory) {
091    this(server, conf, server, (f, p) -> splitLog(f, p, conf, server, sequenceIdChecker, factory));
092  }
093
094  // returns whether we need to continue the split work
095  private static boolean processSyncReplicationWAL(String name, Configuration conf,
096      RegionServerServices server, FileSystem fs, Path walDir) throws IOException {
097    Path walFile = new Path(walDir, name);
098    String filename = walFile.getName();
099    Optional<String> optSyncPeerId =
100      SyncReplicationWALProvider.getSyncReplicationPeerIdFromWALName(filename);
101    if (!optSyncPeerId.isPresent()) {
102      return true;
103    }
104    String peerId = optSyncPeerId.get();
105    ReplicationPeerImpl peer =
106      server.getReplicationSourceService().getReplicationPeers().getPeer(peerId);
107    if (peer == null || !peer.getPeerConfig().isSyncReplication()) {
108      return true;
109    }
110    Pair<SyncReplicationState, SyncReplicationState> stateAndNewState =
111      peer.getSyncReplicationStateAndNewState();
112    if (stateAndNewState.getFirst().equals(SyncReplicationState.ACTIVE) &&
113      stateAndNewState.getSecond().equals(SyncReplicationState.NONE)) {
114      // copy the file to remote and overwrite the previous one
115      String remoteWALDir = peer.getPeerConfig().getRemoteWALDir();
116      Path remoteWALDirForPeer = ReplicationUtils.getPeerRemoteWALDir(remoteWALDir, peerId);
117      Path tmpRemoteWAL = new Path(remoteWALDirForPeer, filename + ".tmp");
118      FileSystem remoteFs = ReplicationUtils.getRemoteWALFileSystem(conf, remoteWALDir);
119      try (FSDataInputStream in = fs.open(walFile);
120        FSDataOutputStream out = remoteFs.createNonRecursive(tmpRemoteWAL, true,
121          CommonFSUtils.getDefaultBufferSize(remoteFs),
122          remoteFs.getDefaultReplication(tmpRemoteWAL), remoteFs.getDefaultBlockSize(tmpRemoteWAL),
123          null)) {
124        IOUtils.copy(in, out);
125      }
126      Path toCommitRemoteWAL =
127        new Path(remoteWALDirForPeer, filename + ReplicationUtils.RENAME_WAL_SUFFIX);
128      // Some FileSystem implementations may not support atomic rename so we need to do it in two
129      // phases
130      FSUtils.renameFile(remoteFs, tmpRemoteWAL, toCommitRemoteWAL);
131      FSUtils.renameFile(remoteFs, toCommitRemoteWAL, new Path(remoteWALDirForPeer, filename));
132    } else if ((stateAndNewState.getFirst().equals(SyncReplicationState.ACTIVE) &&
133      stateAndNewState.getSecond().equals(SyncReplicationState.STANDBY)) ||
134      stateAndNewState.getFirst().equals(SyncReplicationState.STANDBY)) {
135      // check whether we still need to process this file
136      // actually we only write wal file which name is ended with .syncrep in A state, and after
137      // transiting to a state other than A, we will reopen all the regions so the data in the wal
138      // will be flushed so the wal file will be archived soon. But it is still possible that there
139      // is a server crash when we are transiting from A to S, to simplify the logic of the transit
140      // procedure, here we will also check the remote snapshot directory in state S, so that we do
141      // not need wait until all the wal files with .syncrep suffix to be archived before finishing
142      // the procedure.
143      String remoteWALDir = peer.getPeerConfig().getRemoteWALDir();
144      Path remoteSnapshotDirForPeer = ReplicationUtils.getPeerSnapshotWALDir(remoteWALDir, peerId);
145      FileSystem remoteFs = ReplicationUtils.getRemoteWALFileSystem(conf, remoteWALDir);
146      if (remoteFs.exists(new Path(remoteSnapshotDirForPeer, filename))) {
147        // the file has been replayed when the remote cluster was transited from S to DA, the
148        // content will be replicated back to us so give up split it.
149        LOG.warn("Giveup splitting {} since it has been replayed in the remote cluster and " +
150          "the content will be replicated back", filename);
151        return false;
152      }
153    }
154    return true;
155  }
156
157  /**
158   * @return Result either DONE, RESIGNED, or ERR.
159   */
160  static Status splitLog(String filename, CancelableProgressable p, Configuration conf,
161      RegionServerServices server, LastSequenceId sequenceIdChecker, WALFactory factory) {
162    Path walDir;
163    FileSystem fs;
164    try {
165      walDir = CommonFSUtils.getWALRootDir(conf);
166      fs = walDir.getFileSystem(conf);
167    } catch (IOException e) {
168      LOG.warn("Resigning, could not find root dir or fs", e);
169      return Status.RESIGNED;
170    }
171    try {
172      if (!processSyncReplicationWAL(filename, conf, server, fs, walDir)) {
173        return Status.DONE;
174      }
175    } catch (IOException e) {
176      LOG.warn("failed to process sync replication wal {}", filename, e);
177      return Status.RESIGNED;
178    }
179    // TODO have to correctly figure out when log splitting has been
180    // interrupted or has encountered a transient error and when it has
181    // encountered a bad non-retry-able persistent error.
182    try {
183      SplitLogWorkerCoordination splitLogWorkerCoordination =
184         server.getCoordinatedStateManager() == null ? null
185             : server.getCoordinatedStateManager().getSplitLogWorkerCoordination();
186      if (!WALSplitter.splitLogFile(walDir, fs.getFileStatus(new Path(walDir, filename)), fs, conf,
187        p, sequenceIdChecker, splitLogWorkerCoordination, factory, server)) {
188        return Status.PREEMPTED;
189      }
190    } catch (InterruptedIOException iioe) {
191      LOG.warn("Resigning, interrupted splitting WAL {}", filename, iioe);
192      return Status.RESIGNED;
193    } catch (IOException e) {
194      if (e instanceof FileNotFoundException) {
195        // A wal file may not exist anymore. Nothing can be recovered so move on
196        LOG.warn("Done, WAL {} does not exist anymore", filename, e);
197        return Status.DONE;
198      }
199      Throwable cause = e.getCause();
200      if (e instanceof RetriesExhaustedException && (cause instanceof NotServingRegionException
201          || cause instanceof ConnectException || cause instanceof SocketTimeoutException)) {
202        LOG.warn("Resigning, can't connect to target regionserver splitting WAL {}", filename, e);
203        return Status.RESIGNED;
204      } else if (cause instanceof InterruptedException) {
205        LOG.warn("Resigning, interrupted splitting WAL {}", filename, e);
206        return Status.RESIGNED;
207      }
208      LOG.warn("Error splitting WAL {}", filename, e);
209      return Status.ERR;
210    }
211    LOG.debug("Done splitting WAL {}", filename);
212    return Status.DONE;
213  }
214
215  @Override
216  public void run() {
217    try {
218      LOG.info("SplitLogWorker " + server.getServerName() + " starting");
219      coordination.registerListener();
220      // wait for Coordination Engine is ready
221      boolean res = false;
222      while (!res && !coordination.isStop()) {
223        res = coordination.isReady();
224      }
225      if (!coordination.isStop()) {
226        coordination.taskLoop();
227      }
228    } catch (Throwable t) {
229      if (ExceptionUtil.isInterrupt(t)) {
230        LOG.info("SplitLogWorker interrupted. Exiting. " + (coordination.isStop() ? "" :
231            " (ERROR: exitWorker is not set, exiting anyway)"));
232      } else {
233        // only a logical error can cause here. Printing it out
234        // to make debugging easier
235        LOG.error("unexpected error ", t);
236      }
237    } finally {
238      coordination.removeListener();
239      LOG.info("SplitLogWorker " + server.getServerName() + " exiting");
240    }
241  }
242
243  /**
244   * If the worker is doing a task i.e. splitting a log file then stop the task.
245   * It doesn't exit the worker thread.
246   */
247  public void stopTask() {
248    LOG.info("Sending interrupt to stop the worker thread");
249    worker.interrupt(); // TODO interrupt often gets swallowed, do what else?
250  }
251
252  /**
253   * start the SplitLogWorker thread
254   */
255  public void start() {
256    worker = new Thread(null, this, "SplitLogWorker-" + server.getServerName().toShortString());
257    worker.start();
258  }
259
260  /**
261   * stop the SplitLogWorker thread
262   */
263  public void stop() {
264    coordination.stopProcessingTasks();
265    stopTask();
266  }
267
268  /**
269   * Objects implementing this interface actually do the task that has been
270   * acquired by a {@link SplitLogWorker}. Since there isn't a water-tight
271   * guarantee that two workers will not be executing the same task therefore it
272   * is better to have workers prepare the task and then have the
273   * {@link org.apache.hadoop.hbase.master.SplitLogManager} commit the work in
274   * SplitLogManager.TaskFinisher
275   */
276  @FunctionalInterface
277  public interface TaskExecutor {
278    enum Status {
279      DONE(),
280      ERR(),
281      RESIGNED(),
282      PREEMPTED()
283    }
284    Status exec(String name, CancelableProgressable p);
285  }
286
287  /**
288   * Returns the number of tasks processed by coordination.
289   * This method is used by tests only
290   */
291  public int getTaskReadySeq() {
292    return coordination.getTaskReadySeq();
293  }
294}