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.master.procedure;
019
020import java.io.IOException;
021import java.util.Optional;
022import org.apache.hadoop.fs.Path;
023import org.apache.hadoop.hbase.DoNotRetryIOException;
024import org.apache.hadoop.hbase.ServerName;
025import org.apache.hadoop.hbase.procedure2.ProcedureStateSerializer;
026import org.apache.hadoop.hbase.procedure2.RemoteProcedureDispatcher;
027import org.apache.hadoop.hbase.regionserver.SplitWALCallable;
028import org.apache.hadoop.hbase.util.ForeignExceptionUtil;
029import org.apache.hadoop.hbase.wal.AbstractFSWALProvider;
030import org.apache.yetus.audience.InterfaceAudience;
031import org.slf4j.Logger;
032import org.slf4j.LoggerFactory;
033
034import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
035import org.apache.hadoop.hbase.shaded.protobuf.generated.ErrorHandlingProtos;
036import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos;
037
038/**
039 * A remote procedure which is used to send split WAL request to region server. It will return null
040 * if the task succeeded or return a DoNotRetryIOException. {@link SplitWALProcedure} will help
041 * handle the situation that encounters DoNotRetryIOException. Otherwise it will retry until
042 * success.
043 */
044@InterfaceAudience.Private
045public class SplitWALRemoteProcedure extends ServerRemoteProcedure
046  implements ServerProcedureInterface {
047  private static final Logger LOG = LoggerFactory.getLogger(SplitWALRemoteProcedure.class);
048  private String walPath;
049  private ServerName crashedServer;
050
051  public SplitWALRemoteProcedure() {
052  }
053
054  public SplitWALRemoteProcedure(ServerName worker, ServerName crashedServer, String wal) {
055    this.targetServer = worker;
056    this.crashedServer = crashedServer;
057    this.walPath = wal;
058  }
059
060  @Override
061  protected void rollback(MasterProcedureEnv env) throws IOException, InterruptedException {
062    throw new UnsupportedOperationException();
063  }
064
065  @Override
066  protected boolean abort(MasterProcedureEnv env) {
067    return false;
068  }
069
070  @Override
071  protected void serializeStateData(ProcedureStateSerializer serializer) throws IOException {
072    MasterProcedureProtos.SplitWALRemoteData.Builder builder =
073      MasterProcedureProtos.SplitWALRemoteData.newBuilder();
074    builder.setWalPath(walPath).setWorker(ProtobufUtil.toServerName(targetServer))
075      .setCrashedServer(ProtobufUtil.toServerName(crashedServer)).setState(state);
076    if (this.remoteError != null) {
077      ErrorHandlingProtos.ForeignExceptionMessage fem =
078        ForeignExceptionUtil.toProtoForeignException(remoteError);
079      builder.setError(fem);
080    }
081    serializer.serialize(builder.build());
082  }
083
084  @Override
085  protected void deserializeStateData(ProcedureStateSerializer serializer) throws IOException {
086    MasterProcedureProtos.SplitWALRemoteData data =
087      serializer.deserialize(MasterProcedureProtos.SplitWALRemoteData.class);
088    walPath = data.getWalPath();
089    targetServer = ProtobufUtil.toServerName(data.getWorker());
090    crashedServer = ProtobufUtil.toServerName(data.getCrashedServer());
091    state = data.getState();
092    if (data.hasError()) {
093      this.remoteError = ForeignExceptionUtil.toException(data.getError());
094    }
095  }
096
097  @Override
098  public Optional<RemoteProcedureDispatcher.RemoteOperation> remoteCallBuild(MasterProcedureEnv env,
099    ServerName serverName) {
100    return Optional.of(new RSProcedureDispatcher.ServerOperation(
101      this, getProcId(), SplitWALCallable.class, MasterProcedureProtos.SplitWALParameter
102        .newBuilder().setWalPath(walPath).build().toByteArray(),
103      env.getMasterServices().getMasterActiveTime()));
104  }
105
106  @Override
107  protected boolean complete(MasterProcedureEnv env, Throwable error) {
108    if (error == null) {
109      try {
110        env.getMasterServices().getSplitWALManager().archive(walPath);
111      } catch (IOException e) {
112        LOG.warn("Failed split of {}; ignore...", walPath, e);
113      }
114      return true;
115    } else {
116      if (error instanceof DoNotRetryIOException) {
117        LOG.warn("Sent {} to wrong server {}, try another", walPath, targetServer, error);
118        return true;
119      } else {
120        LOG.warn("Failed split of {}, retry...", walPath, error);
121        return false;
122      }
123    }
124  }
125
126  public String getWAL() {
127    return this.walPath;
128  }
129
130  @Override
131  public ServerName getServerName() {
132    // return the crashed server is to use the queue of root ServerCrashProcedure
133    return this.crashedServer;
134  }
135
136  @Override
137  public boolean hasMetaTableRegion() {
138    return AbstractFSWALProvider.isMetaFile(new Path(walPath));
139  }
140
141  @Override
142  public ServerOperationType getServerOperationType() {
143    return ServerOperationType.SPLIT_WAL_REMOTE;
144  }
145
146  @Override
147  protected void toStringClassDetails(StringBuilder builder) {
148    builder.append(getProcName());
149    if (this.targetServer != null) {
150      builder.append(", worker=");
151      builder.append(this.targetServer);
152    }
153  }
154
155  @Override
156  public String getProcName() {
157    return getClass().getSimpleName() + " " + SplitWALProcedure.getWALNameFromStrPath(getWAL());
158  }
159}