View Javadoc

1   /**
2    *
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
10   *
11   *     http://www.apache.org/licenses/LICENSE-2.0
12   *
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.
18   */
19  package org.apache.hadoop.hbase.regionserver;
20  
21  import java.io.FileNotFoundException;
22  import java.io.IOException;
23  import java.io.InterruptedIOException;
24  import java.net.ConnectException;
25  import java.net.SocketTimeoutException;
26  
27  import org.apache.commons.logging.Log;
28  import org.apache.commons.logging.LogFactory;
29  import org.apache.hadoop.hbase.classification.InterfaceAudience;
30  import org.apache.hadoop.conf.Configuration;
31  import org.apache.hadoop.fs.FileSystem;
32  import org.apache.hadoop.fs.Path;
33  import org.apache.hadoop.hbase.NotServingRegionException;
34  import org.apache.hadoop.hbase.Server;
35  import org.apache.hadoop.hbase.client.RetriesExhaustedException;
36  import org.apache.hadoop.hbase.coordination.BaseCoordinatedStateManager;
37  import org.apache.hadoop.hbase.coordination.SplitLogWorkerCoordination;
38  import org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.SplitLogTask.RecoveryMode;
39  import org.apache.hadoop.hbase.wal.WALFactory;
40  import org.apache.hadoop.hbase.wal.WALSplitter;
41  import org.apache.hadoop.hbase.util.CancelableProgressable;
42  import org.apache.hadoop.hbase.util.ExceptionUtil;
43  import org.apache.hadoop.hbase.util.FSUtils;
44  
45  import com.google.common.annotations.VisibleForTesting;
46  
47  /**
48   * This worker is spawned in every regionserver, including master. The Worker waits for log
49   * splitting tasks to be put up by the {@link org.apache.hadoop.hbase.master.SplitLogManager} 
50   * running in the master and races with other workers in other serves to acquire those tasks. 
51   * The coordination is done via coordination engine.
52   * <p>
53   * If a worker has successfully moved the task from state UNASSIGNED to OWNED then it owns the task.
54   * It keeps heart beating the manager by periodically moving the task from UNASSIGNED to OWNED
55   * state. On success it moves the task to TASK_DONE. On unrecoverable error it moves task state to
56   * ERR. If it cannot continue but wants the master to retry the task then it moves the task state to
57   * RESIGNED.
58   * <p>
59   * The manager can take a task away from a worker by moving the task from OWNED to UNASSIGNED. In
60   * the absence of a global lock there is a unavoidable race here - a worker might have just finished
61   * its task when it is stripped of its ownership. Here we rely on the idempotency of the log
62   * splitting task for correctness
63   */
64  @InterfaceAudience.Private
65  public class SplitLogWorker implements Runnable {
66  
67    private static final Log LOG = LogFactory.getLog(SplitLogWorker.class);
68  
69    Thread worker;
70    // thread pool which executes recovery work
71    private SplitLogWorkerCoordination coordination;
72    private Configuration conf;
73    private RegionServerServices server;
74  
75    public SplitLogWorker(Server hserver, Configuration conf, RegionServerServices server,
76        TaskExecutor splitTaskExecutor) {
77      this.server = server;
78      this.conf = conf;
79      this.coordination =
80          ((BaseCoordinatedStateManager) hserver.getCoordinatedStateManager())
81              .getSplitLogWorkerCoordination();
82      this.server = server;
83      coordination.init(server, conf, splitTaskExecutor, this);
84    }
85  
86    public SplitLogWorker(final Server hserver, final Configuration conf,
87        final RegionServerServices server, final LastSequenceId sequenceIdChecker,
88        final WALFactory factory) {
89      this(server, conf, server, new TaskExecutor() {
90        @Override
91        public Status exec(String filename, RecoveryMode mode, CancelableProgressable p) {
92          Path rootdir;
93          FileSystem fs;
94          try {
95            rootdir = FSUtils.getRootDir(conf);
96            fs = rootdir.getFileSystem(conf);
97          } catch (IOException e) {
98            LOG.warn("could not find root dir or fs", e);
99            return Status.RESIGNED;
100         }
101         // TODO have to correctly figure out when log splitting has been
102         // interrupted or has encountered a transient error and when it has
103         // encountered a bad non-retry-able persistent error.
104         try {
105           if (!WALSplitter.splitLogFile(rootdir, fs.getFileStatus(new Path(rootdir, filename)),
106             fs, conf, p, sequenceIdChecker, server.getCoordinatedStateManager(), mode, factory)) {
107             return Status.PREEMPTED;
108           }
109         } catch (InterruptedIOException iioe) {
110           LOG.warn("log splitting of " + filename + " interrupted, resigning", iioe);
111           return Status.RESIGNED;
112         } catch (IOException e) {
113           if (e instanceof FileNotFoundException) {
114             // A wal file may not exist anymore. Nothing can be recovered so move on
115             LOG.warn("WAL " + filename + " does not exist anymore", e);
116             return Status.DONE;
117           }
118           Throwable cause = e.getCause();
119           if (e instanceof RetriesExhaustedException && (cause instanceof NotServingRegionException
120                   || cause instanceof ConnectException
121                   || cause instanceof SocketTimeoutException)) {
122             LOG.warn("log replaying of " + filename + " can't connect to the target regionserver, "
123                 + "resigning", e);
124             return Status.RESIGNED;
125           } else if (cause instanceof InterruptedException) {
126             LOG.warn("log splitting of " + filename + " interrupted, resigning", e);
127             return Status.RESIGNED;
128           }
129           LOG.warn("log splitting of " + filename + " failed, returning error", e);
130           return Status.ERR;
131         }
132         return Status.DONE;
133       }
134     });
135   }
136 
137   @Override
138   public void run() {
139     try {
140       LOG.info("SplitLogWorker " + server.getServerName() + " starting");
141       coordination.registerListener();
142       // wait for Coordination Engine is ready
143       boolean res = false;
144       while (!res && !coordination.isStop()) {
145         res = coordination.isReady();
146       }
147       if (!coordination.isStop()) {
148         coordination.taskLoop();
149       }
150     } catch (Throwable t) {
151       if (ExceptionUtil.isInterrupt(t)) {
152         LOG.info("SplitLogWorker interrupted. Exiting. " + (coordination.isStop() ? "" :
153             " (ERROR: exitWorker is not set, exiting anyway)"));
154       } else {
155         // only a logical error can cause here. Printing it out
156         // to make debugging easier
157         LOG.error("unexpected error ", t);
158       }
159     } finally {
160       coordination.removeListener();
161       LOG.info("SplitLogWorker " + server.getServerName() + " exiting");
162     }
163   }
164 
165   /**
166    * If the worker is doing a task i.e. splitting a log file then stop the task.
167    * It doesn't exit the worker thread.
168    */
169   public void stopTask() {
170     LOG.info("Sending interrupt to stop the worker thread");
171     worker.interrupt(); // TODO interrupt often gets swallowed, do what else?
172   }
173 
174   /**
175    * start the SplitLogWorker thread
176    */
177   public void start() {
178     worker = new Thread(null, this, "SplitLogWorker-" + server.getServerName().toShortString());
179     worker.start();
180   }
181 
182   /**
183    * stop the SplitLogWorker thread
184    */
185   public void stop() {
186     coordination.stopProcessingTasks();
187     stopTask();
188   }
189 
190   /**
191    * Objects implementing this interface actually do the task that has been
192    * acquired by a {@link SplitLogWorker}. Since there isn't a water-tight
193    * guarantee that two workers will not be executing the same task therefore it
194    * is better to have workers prepare the task and then have the
195    * {@link org.apache.hadoop.hbase.master.SplitLogManager} commit the work in 
196    * SplitLogManager.TaskFinisher
197    */
198   public interface TaskExecutor {
199     enum Status {
200       DONE(),
201       ERR(),
202       RESIGNED(),
203       PREEMPTED()
204     }
205     Status exec(String name, RecoveryMode mode, CancelableProgressable p);
206   }
207 
208   /**
209    * Returns the number of tasks processed by coordination.
210    * This method is used by tests only
211    */
212   @VisibleForTesting
213   public int getTaskReadySeq() {
214     return coordination.getTaskReadySeq();
215   }
216 }