|
@@ -31,6 +31,9 @@ import org.apache.hadoop.hdfs.protocolPB.JournalProtocolPB;
|
|
import org.apache.hadoop.hdfs.protocolPB.JournalProtocolServerSideTranslatorPB;
|
|
import org.apache.hadoop.hdfs.protocolPB.JournalProtocolServerSideTranslatorPB;
|
|
import org.apache.hadoop.hdfs.server.common.HdfsServerConstants.NamenodeRole;
|
|
import org.apache.hadoop.hdfs.server.common.HdfsServerConstants.NamenodeRole;
|
|
import org.apache.hadoop.hdfs.server.common.StorageInfo;
|
|
import org.apache.hadoop.hdfs.server.common.StorageInfo;
|
|
|
|
+import org.apache.hadoop.hdfs.server.protocol.FenceResponse;
|
|
|
|
+import org.apache.hadoop.hdfs.server.protocol.FencedException;
|
|
|
|
+import org.apache.hadoop.hdfs.server.protocol.JournalInfo;
|
|
import org.apache.hadoop.hdfs.server.protocol.JournalProtocol;
|
|
import org.apache.hadoop.hdfs.server.protocol.JournalProtocol;
|
|
import org.apache.hadoop.hdfs.server.protocol.NamenodeProtocol;
|
|
import org.apache.hadoop.hdfs.server.protocol.NamenodeProtocol;
|
|
import org.apache.hadoop.hdfs.server.protocol.NamenodeRegistration;
|
|
import org.apache.hadoop.hdfs.server.protocol.NamenodeRegistration;
|
|
@@ -40,6 +43,7 @@ import org.apache.hadoop.ipc.RPC;
|
|
import org.apache.hadoop.net.NetUtils;
|
|
import org.apache.hadoop.net.NetUtils;
|
|
import org.apache.hadoop.security.UserGroupInformation;
|
|
import org.apache.hadoop.security.UserGroupInformation;
|
|
|
|
|
|
|
|
+import com.google.common.annotations.VisibleForTesting;
|
|
import com.google.protobuf.BlockingService;
|
|
import com.google.protobuf.BlockingService;
|
|
|
|
|
|
/**
|
|
/**
|
|
@@ -66,6 +70,8 @@ public class JournalService implements JournalProtocol {
|
|
private final NamenodeProtocol namenode;
|
|
private final NamenodeProtocol namenode;
|
|
private final StateHandler stateHandler = new StateHandler();
|
|
private final StateHandler stateHandler = new StateHandler();
|
|
private final RPC.Server rpcServer;
|
|
private final RPC.Server rpcServer;
|
|
|
|
+ private long epoch = 0;
|
|
|
|
+ private String fencerInfo;
|
|
|
|
|
|
enum State {
|
|
enum State {
|
|
/** The service is initialized and ready to start. */
|
|
/** The service is initialized and ready to start. */
|
|
@@ -115,7 +121,7 @@ public class JournalService implements JournalProtocol {
|
|
current = State.WAITING_FOR_ROLL;
|
|
current = State.WAITING_FOR_ROLL;
|
|
}
|
|
}
|
|
|
|
|
|
- synchronized void startLogSegment() throws IOException {
|
|
|
|
|
|
+ synchronized void startLogSegment() {
|
|
if (current == State.WAITING_FOR_ROLL) {
|
|
if (current == State.WAITING_FOR_ROLL) {
|
|
current = State.SYNCING;
|
|
current = State.SYNCING;
|
|
}
|
|
}
|
|
@@ -232,28 +238,42 @@ public class JournalService implements JournalProtocol {
|
|
}
|
|
}
|
|
|
|
|
|
@Override
|
|
@Override
|
|
- public void journal(NamenodeRegistration registration, long firstTxnId,
|
|
|
|
|
|
+ public void journal(JournalInfo journalInfo, long epoch, long firstTxnId,
|
|
int numTxns, byte[] records) throws IOException {
|
|
int numTxns, byte[] records) throws IOException {
|
|
if (LOG.isTraceEnabled()) {
|
|
if (LOG.isTraceEnabled()) {
|
|
LOG.trace("Received journal " + firstTxnId + " " + numTxns);
|
|
LOG.trace("Received journal " + firstTxnId + " " + numTxns);
|
|
}
|
|
}
|
|
stateHandler.isJournalAllowed();
|
|
stateHandler.isJournalAllowed();
|
|
- verify(registration);
|
|
|
|
|
|
+ verify(epoch, journalInfo);
|
|
listener.journal(this, firstTxnId, numTxns, records);
|
|
listener.journal(this, firstTxnId, numTxns, records);
|
|
}
|
|
}
|
|
|
|
|
|
@Override
|
|
@Override
|
|
- public void startLogSegment(NamenodeRegistration registration, long txid)
|
|
|
|
|
|
+ public void startLogSegment(JournalInfo journalInfo, long epoch, long txid)
|
|
throws IOException {
|
|
throws IOException {
|
|
if (LOG.isTraceEnabled()) {
|
|
if (LOG.isTraceEnabled()) {
|
|
LOG.trace("Received startLogSegment " + txid);
|
|
LOG.trace("Received startLogSegment " + txid);
|
|
}
|
|
}
|
|
stateHandler.isStartLogSegmentAllowed();
|
|
stateHandler.isStartLogSegmentAllowed();
|
|
- verify(registration);
|
|
|
|
|
|
+ verify(epoch, journalInfo);
|
|
listener.rollLogs(this, txid);
|
|
listener.rollLogs(this, txid);
|
|
stateHandler.startLogSegment();
|
|
stateHandler.startLogSegment();
|
|
}
|
|
}
|
|
|
|
|
|
|
|
+ @Override
|
|
|
|
+ public FenceResponse fence(JournalInfo journalInfo, long epoch,
|
|
|
|
+ String fencerInfo) throws IOException {
|
|
|
|
+ LOG.info("Fenced by " + fencerInfo + " with epoch " + epoch);
|
|
|
|
+ verifyFence(epoch, fencerInfo);
|
|
|
|
+ verify(journalInfo);
|
|
|
|
+ long previousEpoch = epoch;
|
|
|
|
+ this.epoch = epoch;
|
|
|
|
+ this.fencerInfo = fencerInfo;
|
|
|
|
+
|
|
|
|
+ // TODO:HDFS-3092 set lastTransId and inSync
|
|
|
|
+ return new FenceResponse(previousEpoch, 0, false);
|
|
|
|
+ }
|
|
|
|
+
|
|
/** Create an RPC server. */
|
|
/** Create an RPC server. */
|
|
private static RPC.Server createRpcServer(Configuration conf,
|
|
private static RPC.Server createRpcServer(Configuration conf,
|
|
InetSocketAddress address, JournalProtocol impl) throws IOException {
|
|
InetSocketAddress address, JournalProtocol impl) throws IOException {
|
|
@@ -267,15 +287,54 @@ public class JournalService implements JournalProtocol {
|
|
address.getHostName(), address.getPort(), 1, false, conf, null);
|
|
address.getHostName(), address.getPort(), 1, false, conf, null);
|
|
}
|
|
}
|
|
|
|
|
|
- private void verify(NamenodeRegistration reg) throws IOException {
|
|
|
|
- if (!registration.getRegistrationID().equals(reg.getRegistrationID())) {
|
|
|
|
- LOG.warn("Invalid registrationID - expected: "
|
|
|
|
- + registration.getRegistrationID() + " received: "
|
|
|
|
- + reg.getRegistrationID());
|
|
|
|
- throw new UnregisteredNodeException(reg);
|
|
|
|
|
|
+ private void verifyEpoch(long e) throws FencedException {
|
|
|
|
+ if (epoch != e) {
|
|
|
|
+ String errorMsg = "Epoch " + e + " is not valid. "
|
|
|
|
+ + "Resource has already been fenced by " + fencerInfo
|
|
|
|
+ + " with epoch " + epoch;
|
|
|
|
+ LOG.warn(errorMsg);
|
|
|
|
+ throw new FencedException(errorMsg);
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ private void verifyFence(long e, String fencer) throws FencedException {
|
|
|
|
+ if (e <= epoch) {
|
|
|
|
+ String errorMsg = "Epoch " + e + " from fencer " + fencer
|
|
|
|
+ + " is not valid. " + "Resource has already been fenced by "
|
|
|
|
+ + fencerInfo + " with epoch " + epoch;
|
|
|
|
+ LOG.warn(errorMsg);
|
|
|
|
+ throw new FencedException(errorMsg);
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * Verifies a journal request
|
|
|
|
+ */
|
|
|
|
+ private void verify(JournalInfo journalInfo) throws IOException {
|
|
|
|
+ String errorMsg = null;
|
|
|
|
+ int expectedNamespaceID = registration.getNamespaceID();
|
|
|
|
+ if (journalInfo.getNamespaceId() != expectedNamespaceID) {
|
|
|
|
+ errorMsg = "Invalid namespaceID in journal request - expected " + expectedNamespaceID
|
|
|
|
+ + " actual " + journalInfo.getNamespaceId();
|
|
|
|
+ LOG.warn(errorMsg);
|
|
|
|
+ throw new UnregisteredNodeException(journalInfo);
|
|
|
|
+ }
|
|
|
|
+ if (!journalInfo.getClusterId().equals(registration.getClusterID())) {
|
|
|
|
+ errorMsg = "Invalid clusterId in journal request - expected "
|
|
|
|
+ + journalInfo.getClusterId() + " actual " + registration.getClusterID();
|
|
|
|
+ LOG.warn(errorMsg);
|
|
|
|
+ throw new UnregisteredNodeException(journalInfo);
|
|
}
|
|
}
|
|
}
|
|
}
|
|
|
|
|
|
|
|
+ /**
|
|
|
|
+ * Verifies a journal request
|
|
|
|
+ */
|
|
|
|
+ private void verify(long e, JournalInfo journalInfo) throws IOException {
|
|
|
|
+ verifyEpoch(e);
|
|
|
|
+ verify(journalInfo);
|
|
|
|
+ }
|
|
|
|
+
|
|
/**
|
|
/**
|
|
* Register this service with the active namenode.
|
|
* Register this service with the active namenode.
|
|
*/
|
|
*/
|
|
@@ -298,4 +357,9 @@ public class JournalService implements JournalProtocol {
|
|
listener.verifyVersion(this, nsInfo);
|
|
listener.verifyVersion(this, nsInfo);
|
|
registration.setStorageInfo(nsInfo);
|
|
registration.setStorageInfo(nsInfo);
|
|
}
|
|
}
|
|
-}
|
|
|
|
|
|
+
|
|
|
|
+ @VisibleForTesting
|
|
|
|
+ long getEpoch() {
|
|
|
|
+ return epoch;
|
|
|
|
+ }
|
|
|
|
+}
|