HDFS-10687. Federation Membership State Store internal API. Contributed by Jason Kace and Inigo Goiri.

This commit is contained in:
Inigo Goiri 2017-07-31 10:55:21 -07:00
parent 47db6e9d8e
commit 95cae08849
42 changed files with 4745 additions and 32 deletions

View File

@ -14,6 +14,9 @@
<Match>
<Package name="org.apache.hadoop.hdfs.qjournal.protocol" />
</Match>
<Match>
<Package name="org.apache.hadoop.hdfs.federation.protocol.proto" />
</Match>
<Match>
<Bug pattern="EI_EXPOSE_REP" />
</Match>

View File

@ -331,6 +331,7 @@ http://maven.apache.org/xsd/maven-4.0.0.xsd">
<include>QJournalProtocol.proto</include>
<include>editlog.proto</include>
<include>fsimage.proto</include>
<include>FederationProtocol.proto</include>
</includes>
</source>
</configuration>

View File

@ -27,6 +27,8 @@ import org.apache.hadoop.hdfs.protocol.HdfsConstants;
import org.apache.hadoop.hdfs.server.blockmanagement.BlockPlacementPolicyDefault;
import org.apache.hadoop.hdfs.server.blockmanagement.BlockPlacementPolicyRackFaultTolerant;
import org.apache.hadoop.hdfs.server.datanode.fsdataset.impl.RamDiskReplicaLruTracker;
import org.apache.hadoop.hdfs.server.federation.resolver.ActiveNamenodeResolver;
import org.apache.hadoop.hdfs.server.federation.resolver.MembershipNamenodeResolver;
import org.apache.hadoop.hdfs.server.federation.store.driver.StateStoreDriver;
import org.apache.hadoop.hdfs.server.federation.store.driver.impl.StateStoreFileImpl;
import org.apache.hadoop.hdfs.server.federation.store.driver.impl.StateStoreSerializerPBImpl;
@ -1169,8 +1171,9 @@ public class DFSConfigKeys extends CommonConfigurationKeys {
"org.apache.hadoop.hdfs.server.federation.MockResolver";
public static final String FEDERATION_NAMENODE_RESOLVER_CLIENT_CLASS =
FEDERATION_ROUTER_PREFIX + "namenode.resolver.client.class";
public static final String FEDERATION_NAMENODE_RESOLVER_CLIENT_CLASS_DEFAULT =
"org.apache.hadoop.hdfs.server.federation.MockResolver";
public static final Class<? extends ActiveNamenodeResolver>
FEDERATION_NAMENODE_RESOLVER_CLIENT_CLASS_DEFAULT =
MembershipNamenodeResolver.class;
// HDFS Router-based federation State Store
public static final String FEDERATION_STORE_PREFIX =
@ -1192,6 +1195,16 @@ public class DFSConfigKeys extends CommonConfigurationKeys {
public static final long FEDERATION_STORE_CONNECTION_TEST_MS_DEFAULT =
TimeUnit.MINUTES.toMillis(1);
public static final String DFS_ROUTER_CACHE_TIME_TO_LIVE_MS =
FEDERATION_ROUTER_PREFIX + "cache.ttl";
public static final long DFS_ROUTER_CACHE_TIME_TO_LIVE_MS_DEFAULT =
TimeUnit.MINUTES.toMillis(1);
public static final String FEDERATION_STORE_MEMBERSHIP_EXPIRATION_MS =
FEDERATION_STORE_PREFIX + "membership.expiration";
public static final long FEDERATION_STORE_MEMBERSHIP_EXPIRATION_MS_DEFAULT =
TimeUnit.MINUTES.toMillis(5);
// dfs.client.retry confs are moved to HdfsClientConfigKeys.Retry
@Deprecated
public static final String DFS_CLIENT_RETRY_POLICY_ENABLED_KEY

View File

@ -0,0 +1,290 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.resolver;
import static org.apache.hadoop.hdfs.server.federation.resolver.FederationNamenodeServiceState.ACTIVE;
import static org.apache.hadoop.hdfs.server.federation.resolver.FederationNamenodeServiceState.EXPIRED;
import static org.apache.hadoop.hdfs.server.federation.resolver.FederationNamenodeServiceState.UNAVAILABLE;
import java.io.IOException;
import java.net.InetSocketAddress;
import java.util.ArrayList;
import java.util.Collections;
import java.util.Iterator;
import java.util.List;
import java.util.Map;
import java.util.Set;
import java.util.concurrent.ConcurrentHashMap;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hdfs.server.federation.store.MembershipStore;
import org.apache.hadoop.hdfs.server.federation.store.StateStoreCache;
import org.apache.hadoop.hdfs.server.federation.store.StateStoreService;
import org.apache.hadoop.hdfs.server.federation.store.StateStoreUnavailableException;
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetNamenodeRegistrationsRequest;
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetNamenodeRegistrationsResponse;
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetNamespaceInfoRequest;
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetNamespaceInfoResponse;
import org.apache.hadoop.hdfs.server.federation.store.protocol.NamenodeHeartbeatRequest;
import org.apache.hadoop.hdfs.server.federation.store.protocol.UpdateNamenodeRegistrationRequest;
import org.apache.hadoop.hdfs.server.federation.store.records.MembershipState;
import org.apache.hadoop.util.Time;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
/**
* Implements a cached lookup of the most recently active namenode for a
* particular nameservice. Relies on the {@link StateStoreService} to
* discover available nameservices and namenodes.
*/
public class MembershipNamenodeResolver
implements ActiveNamenodeResolver, StateStoreCache {
private static final Logger LOG =
LoggerFactory.getLogger(MembershipNamenodeResolver.class);
/** Reference to the State Store. */
private final StateStoreService stateStore;
/** Membership State Store interface. */
private final MembershipStore membershipInterface;
/** Parent router ID. */
private String routerId;
/** Cached lookup of NN for nameservice. Invalidated on cache refresh. */
private Map<String, List<? extends FederationNamenodeContext>> cacheNS;
/** Cached lookup of NN for block pool. Invalidated on cache refresh. */
private Map<String, List<? extends FederationNamenodeContext>> cacheBP;
public MembershipNamenodeResolver(
Configuration conf, StateStoreService store) throws IOException {
this.stateStore = store;
this.cacheNS = new ConcurrentHashMap<>();
this.cacheBP = new ConcurrentHashMap<>();
if (this.stateStore != null) {
// Request cache updates from the state store
this.stateStore.registerCacheExternal(this);
// Initialize the interface to get the membership
this.membershipInterface = this.stateStore.getRegisteredRecordStore(
MembershipStore.class);
} else {
this.membershipInterface = null;
}
if (this.membershipInterface == null) {
throw new IOException("State Store does not have an interface for " +
MembershipStore.class.getSimpleName());
}
}
@Override
public boolean loadCache(boolean force) {
// Our cache depends on the store, update it first
try {
this.membershipInterface.loadCache(force);
} catch (IOException e) {
LOG.error("Cannot update membership from the State Store", e);
}
// Force refresh of active NN cache
cacheBP.clear();
cacheNS.clear();
return true;
}
@Override
public void updateActiveNamenode(
final String nsId, final InetSocketAddress address) throws IOException {
// Called when we have an RPC miss and successful hit on an alternate NN.
// Temporarily update our cache, it will be overwritten on the next update.
try {
MembershipState partial = MembershipState.newInstance();
String rpcAddress = address.getHostName() + ":" + address.getPort();
partial.setRpcAddress(rpcAddress);
partial.setNameserviceId(nsId);
GetNamenodeRegistrationsRequest request =
GetNamenodeRegistrationsRequest.newInstance(partial);
GetNamenodeRegistrationsResponse response =
this.membershipInterface.getNamenodeRegistrations(request);
List<MembershipState> records = response.getNamenodeMemberships();
if (records != null && records.size() == 1) {
MembershipState record = records.get(0);
UpdateNamenodeRegistrationRequest updateRequest =
UpdateNamenodeRegistrationRequest.newInstance(
record.getNameserviceId(), record.getNamenodeId(), ACTIVE);
this.membershipInterface.updateNamenodeRegistration(updateRequest);
}
} catch (StateStoreUnavailableException e) {
LOG.error("Cannot update {} as active, State Store unavailable", address);
}
}
@Override
public List<? extends FederationNamenodeContext> getNamenodesForNameserviceId(
final String nsId) throws IOException {
List<? extends FederationNamenodeContext> ret = cacheNS.get(nsId);
if (ret == null) {
try {
MembershipState partial = MembershipState.newInstance();
partial.setNameserviceId(nsId);
GetNamenodeRegistrationsRequest request =
GetNamenodeRegistrationsRequest.newInstance(partial);
final List<MembershipState> result =
getRecentRegistrationForQuery(request, true, false);
if (result == null || result.isEmpty()) {
LOG.error("Cannot locate eligible NNs for {}", nsId);
return null;
} else {
cacheNS.put(nsId, result);
ret = result;
}
} catch (StateStoreUnavailableException e) {
LOG.error("Cannot get active NN for {}, State Store unavailable", nsId);
}
}
if (ret == null) {
return null;
}
return Collections.unmodifiableList(ret);
}
@Override
public List<? extends FederationNamenodeContext> getNamenodesForBlockPoolId(
final String bpId) throws IOException {
List<? extends FederationNamenodeContext> ret = cacheBP.get(bpId);
if (ret == null) {
try {
MembershipState partial = MembershipState.newInstance();
partial.setBlockPoolId(bpId);
GetNamenodeRegistrationsRequest request =
GetNamenodeRegistrationsRequest.newInstance(partial);
final List<MembershipState> result =
getRecentRegistrationForQuery(request, true, false);
if (result == null || result.isEmpty()) {
LOG.error("Cannot locate eligible NNs for {}", bpId);
} else {
cacheBP.put(bpId, result);
ret = result;
}
} catch (StateStoreUnavailableException e) {
LOG.error("Cannot get active NN for {}, State Store unavailable", bpId);
return null;
}
}
if (ret == null) {
return null;
}
return Collections.unmodifiableList(ret);
}
@Override
public boolean registerNamenode(NamenodeStatusReport report)
throws IOException {
if (this.routerId == null) {
LOG.warn("Cannot register namenode, router ID is not known {}", report);
return false;
}
MembershipState record = MembershipState.newInstance(
routerId, report.getNameserviceId(), report.getNamenodeId(),
report.getClusterId(), report.getBlockPoolId(), report.getRpcAddress(),
report.getServiceAddress(), report.getLifelineAddress(),
report.getWebAddress(), report.getState(), report.getSafemode());
if (report.getState() != UNAVAILABLE) {
// Set/update our last contact time
record.setLastContact(Time.now());
}
NamenodeHeartbeatRequest request = NamenodeHeartbeatRequest.newInstance();
request.setNamenodeMembership(record);
return this.membershipInterface.namenodeHeartbeat(request).getResult();
}
@Override
public Set<FederationNamespaceInfo> getNamespaces() throws IOException {
GetNamespaceInfoRequest request = GetNamespaceInfoRequest.newInstance();
GetNamespaceInfoResponse response =
this.membershipInterface.getNamespaceInfo(request);
return response.getNamespaceInfo();
}
/**
* Picks the most relevant record registration that matches the query. Return
* registrations matching the query in this preference: 1) Most recently
* updated ACTIVE registration 2) Most recently updated STANDBY registration
* (if showStandby) 3) Most recently updated UNAVAILABLE registration (if
* showUnavailable). EXPIRED registrations are ignored.
*
* @param query The select query for NN registrations.
* @param excludes List of NNs to exclude from matching results.
* @param addUnavailable include UNAVAILABLE registrations.
* @param addExpired include EXPIRED registrations.
* @return List of memberships or null if no registrations that
* both match the query AND the selected states.
* @throws IOException
*/
private List<MembershipState> getRecentRegistrationForQuery(
GetNamenodeRegistrationsRequest request, boolean addUnavailable,
boolean addExpired) throws IOException {
// Retrieve a list of all registrations that match this query.
// This may include all NN records for a namespace/blockpool, including
// duplicate records for the same NN from different routers.
GetNamenodeRegistrationsResponse response =
this.membershipInterface.getNamenodeRegistrations(request);
List<MembershipState> memberships = response.getNamenodeMemberships();
if (!addExpired || !addUnavailable) {
Iterator<MembershipState> iterator = memberships.iterator();
while (iterator.hasNext()) {
MembershipState membership = iterator.next();
if (membership.getState() == EXPIRED && !addExpired) {
iterator.remove();
} else if (membership.getState() == UNAVAILABLE && !addUnavailable) {
iterator.remove();
}
}
}
List<MembershipState> priorityList = new ArrayList<>();
priorityList.addAll(memberships);
Collections.sort(priorityList, new NamenodePriorityComparator());
LOG.debug("Selected most recent NN {} for query", priorityList);
return priorityList;
}
@Override
public void setRouterId(String router) {
this.routerId = router;
}
}

View File

@ -19,25 +19,56 @@ package org.apache.hadoop.hdfs.server.federation.router;
import java.lang.reflect.Constructor;
import org.apache.commons.logging.Log;
import org.apache.commons.logging.LogFactory;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hdfs.DFSConfigKeys;
import org.apache.hadoop.hdfs.server.federation.resolver.ActiveNamenodeResolver;
import org.apache.hadoop.hdfs.server.federation.resolver.FileSubclusterResolver;
import org.apache.hadoop.hdfs.server.federation.store.StateStoreService;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
/**
* Utilities for managing HDFS federation.
*/
public final class FederationUtil {
private static final Log LOG = LogFactory.getLog(FederationUtil.class);
private static final Logger LOG =
LoggerFactory.getLogger(FederationUtil.class);
private FederationUtil() {
// Utility Class
}
/**
* Create an instance of an interface with a constructor using a context.
*
* @param conf Configuration for the class names.
* @param context Context object to pass to the instance.
* @param contextClass Type of the context passed to the constructor.
* @param clazz Class of the object to return.
* @return New instance of the specified class that implements the desired
* interface and a single parameter constructor containing a
* StateStore reference.
*/
private static <T, R> T newInstance(final Configuration conf,
final R context, final Class<R> contextClass, final Class<T> clazz) {
try {
if (contextClass == null) {
// Default constructor if no context
Constructor<T> constructor = clazz.getConstructor();
return constructor.newInstance();
} else {
// Constructor with context
Constructor<T> constructor = clazz.getConstructor(
Configuration.class, contextClass);
return constructor.newInstance(conf, context);
}
} catch (ReflectiveOperationException e) {
LOG.error("Could not instantiate: {}", clazz.getSimpleName(), e);
return null;
}
}
/**
* Create an instance of an interface with a constructor using a state store
* constructor.
@ -105,13 +136,14 @@ public final class FederationUtil {
*
* @param conf Configuration that defines the namenode resolver class.
* @param obj Context object passed to class constructor.
* @return ActiveNamenodeResolver
* @return New active namenode resolver.
*/
public static ActiveNamenodeResolver newActiveNamenodeResolver(
Configuration conf, StateStoreService stateStore) {
return newInstance(conf, stateStore, StateStoreService.class,
Class<? extends ActiveNamenodeResolver> clazz = conf.getClass(
DFSConfigKeys.FEDERATION_NAMENODE_RESOLVER_CLIENT_CLASS,
DFSConfigKeys.FEDERATION_NAMENODE_RESOLVER_CLIENT_CLASS_DEFAULT,
ActiveNamenodeResolver.class);
return newInstance(conf, stateStore, StateStoreService.class, clazz);
}
}

View File

@ -0,0 +1,237 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store;
import java.io.IOException;
import java.util.ArrayList;
import java.util.Collections;
import java.util.LinkedList;
import java.util.List;
import java.util.concurrent.locks.Lock;
import java.util.concurrent.locks.ReadWriteLock;
import java.util.concurrent.locks.ReentrantReadWriteLock;
import org.apache.hadoop.hdfs.server.federation.store.driver.StateStoreDriver;
import org.apache.hadoop.hdfs.server.federation.store.records.BaseRecord;
import org.apache.hadoop.hdfs.server.federation.store.records.QueryResult;
import org.apache.hadoop.util.Time;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
/**
* Record store that takes care of caching the records in memory.
*
* @param <R> Record to store by this interface.
*/
public abstract class CachedRecordStore<R extends BaseRecord>
extends RecordStore<R> implements StateStoreCache {
private static final Logger LOG =
LoggerFactory.getLogger(CachedRecordStore.class);
/** Prevent loading the cache more than once every 500 ms. */
private static final long MIN_UPDATE_MS = 500;
/** Cached entries. */
private List<R> records = new ArrayList<>();
/** Time stamp of the cached entries. */
private long timestamp = -1;
/** If the cache is initialized. */
private boolean initialized = false;
/** Last time the cache was updated. */
private long lastUpdate = -1;
/** Lock to access the memory cache. */
private final ReadWriteLock readWriteLock = new ReentrantReadWriteLock();
private final Lock readLock = readWriteLock.readLock();
private final Lock writeLock = readWriteLock.writeLock();
/** If it should override the expired values when loading the cache. */
private boolean override = false;
/**
* Create a new cached record store.
*
* @param clazz Class of the record to store.
* @param driver State Store driver.
*/
protected CachedRecordStore(Class<R> clazz, StateStoreDriver driver) {
this(clazz, driver, false);
}
/**
* Create a new cached record store.
*
* @param clazz Class of the record to store.
* @param driver State Store driver.
* @param override If the entries should be override if they expire
*/
protected CachedRecordStore(
Class<R> clazz, StateStoreDriver driver, boolean over) {
super(clazz, driver);
this.override = over;
}
/**
* Check that the cache of the State Store information is available.
*
* @throws StateStoreUnavailableException If the cache is not initialized.
*/
private void checkCacheAvailable() throws StateStoreUnavailableException {
if (!this.initialized) {
throw new StateStoreUnavailableException(
"Cached State Store not initialized, " +
getRecordClass().getSimpleName() + " records not valid");
}
}
@Override
public boolean loadCache(boolean force) throws IOException {
// Prevent loading the cache too frequently
if (force || isUpdateTime()) {
List<R> newRecords = null;
long t = -1;
try {
QueryResult<R> result = getDriver().get(getRecordClass());
newRecords = result.getRecords();
t = result.getTimestamp();
// If we have any expired record, update the State Store
if (this.override) {
overrideExpiredRecords(result);
}
} catch (IOException e) {
LOG.error("Cannot get \"{}\" records from the State Store",
getRecordClass().getSimpleName());
this.initialized = false;
return false;
}
// Update cache atomically
writeLock.lock();
try {
this.records.clear();
this.records.addAll(newRecords);
this.timestamp = t;
this.initialized = true;
} finally {
writeLock.unlock();
}
lastUpdate = Time.monotonicNow();
}
return true;
}
/**
* Check if it's time to update the cache. Update it it was never updated.
*
* @return If it's time to update this cache.
*/
private boolean isUpdateTime() {
return Time.monotonicNow() - lastUpdate > MIN_UPDATE_MS;
}
/**
* Updates the state store with any record overrides we detected, such as an
* expired state.
*
* @param query RecordQueryResult containing the data to be inspected.
* @param clazz Type of objects contained in the query.
* @throws IOException
*/
public void overrideExpiredRecords(QueryResult<R> query) throws IOException {
List<R> commitRecords = new ArrayList<>();
List<R> newRecords = query.getRecords();
long currentDriverTime = query.getTimestamp();
if (newRecords == null || currentDriverTime <= 0) {
LOG.error("Cannot check overrides for record");
return;
}
for (R record : newRecords) {
if (record.checkExpired(currentDriverTime)) {
String recordName = StateStoreUtils.getRecordName(record.getClass());
LOG.info("Override State Store record {}: {}", recordName, record);
commitRecords.add(record);
}
}
if (commitRecords.size() > 0) {
getDriver().putAll(commitRecords, true, false);
}
}
/**
* Updates the state store with any record overrides we detected, such as an
* expired state.
*
* @param driver State store driver for the data store.
* @param record Record record to be updated.
* @param clazz Type of data record.
* @throws IOException
*/
public void overrideExpiredRecord(R record) throws IOException {
List<R> newRecords = Collections.singletonList(record);
long time = getDriver().getTime();
QueryResult<R> query = new QueryResult<>(newRecords, time);
overrideExpiredRecords(query);
}
/**
* Get all the cached records.
*
* @return Copy of the cached records.
* @throws StateStoreUnavailableException If the State store is not available.
*/
public List<R> getCachedRecords() throws StateStoreUnavailableException {
checkCacheAvailable();
List<R> ret = new LinkedList<R>();
this.readLock.lock();
try {
ret.addAll(this.records);
} finally {
this.readLock.unlock();
}
return ret;
}
/**
* Get all the cached records and the time stamp of the cache.
*
* @return Copy of the cached records and the time stamp.
* @throws StateStoreUnavailableException If the State store is not available.
*/
protected QueryResult<R> getCachedRecordsAndTimeStamp()
throws StateStoreUnavailableException {
checkCacheAvailable();
this.readLock.lock();
try {
return new QueryResult<R>(this.records, this.timestamp);
} finally {
this.readLock.unlock();
}
}
}

View File

@ -0,0 +1,126 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store;
import java.io.IOException;
import org.apache.hadoop.classification.InterfaceAudience;
import org.apache.hadoop.classification.InterfaceStability;
import org.apache.hadoop.hdfs.server.federation.store.driver.StateStoreDriver;
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetNamenodeRegistrationsRequest;
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetNamenodeRegistrationsResponse;
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetNamespaceInfoRequest;
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetNamespaceInfoResponse;
import org.apache.hadoop.hdfs.server.federation.store.protocol.NamenodeHeartbeatRequest;
import org.apache.hadoop.hdfs.server.federation.store.protocol.NamenodeHeartbeatResponse;
import org.apache.hadoop.hdfs.server.federation.store.protocol.UpdateNamenodeRegistrationRequest;
import org.apache.hadoop.hdfs.server.federation.store.protocol.UpdateNamenodeRegistrationResponse;
import org.apache.hadoop.hdfs.server.federation.store.records.MembershipState;
/**
* Management API for NameNode registrations stored in
* {@link org.apache.hadoop.hdfs.server.federation.store.records.MembershipState
* MembershipState} records. The {@link org.apache.hadoop.hdfs.server.
* federation.router.RouterHeartbeatService RouterHeartbeatService} periodically
* polls each NN to update the NameNode metadata(addresses, operational) and HA
* state(active, standby). Each NameNode may be polled by multiple
* {@link org.apache.hadoop.hdfs.server.federation.router.Router Router}
* instances.
* <p>
* Once fetched from the
* {@link org.apache.hadoop.hdfs.server.federation.store.driver.StateStoreDriver
* StateStoreDriver}, NameNode registrations are cached until the next query.
* The fetched registration data is aggregated using a quorum to determine the
* best/most accurate state for each NameNode. The cache is periodically updated
* by the @{link StateStoreCacheUpdateService}.
*/
@InterfaceAudience.Private
@InterfaceStability.Evolving
public abstract class MembershipStore
extends CachedRecordStore<MembershipState> {
protected MembershipStore(StateStoreDriver driver) {
super(MembershipState.class, driver, true);
}
/**
* Inserts or updates a namenode membership entry into the table.
*
* @param request Fully populated NamenodeHeartbeatRequest request.
* @return True if successful, false otherwise.
* @throws StateStoreUnavailableException Throws exception if the data store
* is not initialized.
* @throws IOException if the data store could not be queried or the query is
* invalid.
*/
public abstract NamenodeHeartbeatResponse namenodeHeartbeat(
NamenodeHeartbeatRequest request) throws IOException;
/**
* Queries for a single cached registration entry matching the given
* parameters. Possible keys are the names of data structure elements Possible
* values are matching SQL "LIKE" targets.
*
* @param request Fully populated GetNamenodeRegistrationsRequest request.
* @return Single matching FederationMembershipStateEntry or null if not found
* or more than one entry matches.
* @throws StateStoreUnavailableException Throws exception if the data store
* is not initialized.
* @throws IOException if the data store could not be queried or the query is
* invalid.
*/
public abstract GetNamenodeRegistrationsResponse getNamenodeRegistrations(
GetNamenodeRegistrationsRequest request) throws IOException;
/**
* Get the expired registrations from the registration cache.
*
* @return Expired registrations or zero-length list if none are found.
* @throws StateStoreUnavailableException Throws exception if the data store
* is not initialized.
* @throws IOException if the data store could not be queried or the query is
* invalid.
*/
public abstract GetNamenodeRegistrationsResponse
getExpiredNamenodeRegistrations(GetNamenodeRegistrationsRequest request)
throws IOException;
/**
* Retrieves a list of registered nameservices and their associated info.
*
* @param request
* @return Collection of information for each registered nameservice.
* @throws IOException if the data store could not be queried or the query is
* invalid.
*/
public abstract GetNamespaceInfoResponse getNamespaceInfo(
GetNamespaceInfoRequest request) throws IOException;
/**
* Overrides a cached namenode state with an updated state.
*
* @param request Fully populated OverrideNamenodeRegistrationRequest request.
* @return OverrideNamenodeRegistrationResponse
* @throws StateStoreUnavailableException if the data store is not
* initialized.
* @throws IOException if the data store could not be queried or the query is
* invalid.
*/
public abstract UpdateNamenodeRegistrationResponse updateNamenodeRegistration(
UpdateNamenodeRegistrationRequest request) throws IOException;
}

View File

@ -0,0 +1,36 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store;
import java.io.IOException;
/**
* Interface for a cached copy of the State Store.
*/
public interface StateStoreCache {
/**
* Load the cache from the State Store. Called by the cache update service
* when the data has been reloaded.
*
* @param force If we force the load.
* @return If the cache was loaded successfully.
* @throws IOException If there was an error loading the cache.
*/
boolean loadCache(boolean force) throws IOException;
}

View File

@ -0,0 +1,67 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hdfs.DFSConfigKeys;
import org.apache.hadoop.hdfs.server.federation.router.PeriodicService;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
/**
* Service to periodically update the {@link StateStoreService}
* cached information in the
* {@link org.apache.hadoop.hdfs.server.federation.router.Router Router}.
* This is for performance and removes the State Store from the critical path
* in common operations.
*/
public class StateStoreCacheUpdateService extends PeriodicService {
private static final Logger LOG =
LoggerFactory.getLogger(StateStoreCacheUpdateService.class);
/** The service that manages the State Store connection. */
private final StateStoreService stateStore;
/**
* Create a new Cache update service.
*
* @param stateStore Implementation of the state store
*/
public StateStoreCacheUpdateService(StateStoreService stateStore) {
super(StateStoreCacheUpdateService.class.getSimpleName());
this.stateStore = stateStore;
}
@Override
protected void serviceInit(Configuration conf) throws Exception {
this.setIntervalMs(conf.getLong(
DFSConfigKeys.DFS_ROUTER_CACHE_TIME_TO_LIVE_MS,
DFSConfigKeys.DFS_ROUTER_CACHE_TIME_TO_LIVE_MS_DEFAULT));
super.serviceInit(conf);
}
@Override
public void periodicInvoke() {
LOG.debug("Updating State Store cache");
stateStore.refreshCaches();
}
}

View File

@ -18,17 +18,24 @@
package org.apache.hadoop.hdfs.server.federation.store;
import java.io.IOException;
import java.util.ArrayList;
import java.util.Collection;
import java.util.HashMap;
import java.util.LinkedList;
import java.util.List;
import java.util.Map;
import org.apache.hadoop.classification.InterfaceAudience;
import org.apache.hadoop.classification.InterfaceStability;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hdfs.DFSConfigKeys;
import org.apache.hadoop.hdfs.server.federation.store.driver.StateStoreDriver;
import org.apache.hadoop.hdfs.server.federation.store.impl.MembershipStoreImpl;
import org.apache.hadoop.hdfs.server.federation.store.records.BaseRecord;
import org.apache.hadoop.hdfs.server.federation.store.records.MembershipState;
import org.apache.hadoop.service.CompositeService;
import org.apache.hadoop.util.ReflectionUtils;
import org.apache.hadoop.util.Time;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
@ -54,6 +61,9 @@ import com.google.common.annotations.VisibleForTesting;
* federation.
* <li>{@link MountTableStore}: Mount table between to subclusters.
* See {@link org.apache.hadoop.fs.viewfs.ViewFs ViewFs}.
* <li>{@link RebalancerStore}: Log of the rebalancing operations.
* <li>{@link RouterStore}: Router state in the federation.
* <li>{@link TokenStore}: Tokens in the federation.
* </ul>
*/
@InterfaceAudience.Private
@ -77,8 +87,30 @@ public class StateStoreService extends CompositeService {
private StateStoreConnectionMonitorService monitorService;
/** Supported record stores. */
private final Map<
Class<? extends BaseRecord>, RecordStore<? extends BaseRecord>>
recordStores;
/** Service to maintain State Store caches. */
private StateStoreCacheUpdateService cacheUpdater;
/** Time the cache was last successfully updated. */
private long cacheLastUpdateTime;
/** List of internal caches to update. */
private final List<StateStoreCache> cachesToUpdateInternal;
/** List of external caches to update. */
private final List<StateStoreCache> cachesToUpdateExternal;
public StateStoreService() {
super(StateStoreService.class.getName());
// Records and stores supported by this implementation
this.recordStores = new HashMap<>();
// Caches to maintain
this.cachesToUpdateInternal = new ArrayList<>();
this.cachesToUpdateExternal = new ArrayList<>();
}
/**
@ -102,10 +134,22 @@ public class StateStoreService extends CompositeService {
throw new IOException("Cannot create driver for the State Store");
}
// Add supported record stores
addRecordStore(MembershipStoreImpl.class);
// Check the connection to the State Store periodically
this.monitorService = new StateStoreConnectionMonitorService(this);
this.addService(monitorService);
// Set expirations intervals for each record
MembershipState.setExpirationMs(conf.getLong(
DFSConfigKeys.FEDERATION_STORE_MEMBERSHIP_EXPIRATION_MS,
DFSConfigKeys.FEDERATION_STORE_MEMBERSHIP_EXPIRATION_MS_DEFAULT));
// Cache update service
this.cacheUpdater = new StateStoreCacheUpdateService(this);
addService(this.cacheUpdater);
super.serviceInit(this.conf);
}
@ -122,14 +166,57 @@ public class StateStoreService extends CompositeService {
super.serviceStop();
}
/**
* Add a record store to the State Store. It includes adding the store, the
* supported record and the cache management.
*
* @param clazz Class of the record store to track.
* @return New record store.
* @throws ReflectiveOperationException
*/
private <T extends RecordStore<?>> void addRecordStore(
final Class<T> clazz) throws ReflectiveOperationException {
assert this.getServiceState() == STATE.INITED :
"Cannot add record to the State Store once started";
T recordStore = RecordStore.newInstance(clazz, this.getDriver());
Class<? extends BaseRecord> recordClass = recordStore.getRecordClass();
this.recordStores.put(recordClass, recordStore);
// Subscribe for cache updates
if (recordStore instanceof StateStoreCache) {
StateStoreCache cachedRecordStore = (StateStoreCache) recordStore;
this.cachesToUpdateInternal.add(cachedRecordStore);
}
}
/**
* Get the record store in this State Store for a given interface.
*
* @param recordStoreClass Class of the record store.
* @return Registered record store or null if not found.
*/
public <T extends RecordStore<?>> T getRegisteredRecordStore(
final Class<T> recordStoreClass) {
for (RecordStore<? extends BaseRecord> recordStore :
this.recordStores.values()) {
if (recordStoreClass.isInstance(recordStore)) {
@SuppressWarnings("unchecked")
T recordStoreChecked = (T) recordStore;
return recordStoreChecked;
}
}
return null;
}
/**
* List of records supported by this State Store.
*
* @return List of supported record classes.
*/
public Collection<Class<? extends BaseRecord>> getSupportedRecords() {
// TODO add list of records
return new LinkedList<>();
return this.recordStores.keySet();
}
/**
@ -142,6 +229,7 @@ public class StateStoreService extends CompositeService {
if (this.driver.init(conf, getIdentifier(), getSupportedRecords())) {
LOG.info("Connection to the State Store driver {} is open and ready",
driverName);
this.refreshCaches();
} else {
LOG.error("Cannot initialize State Store driver {}", driverName);
}
@ -198,4 +286,114 @@ public class StateStoreService extends CompositeService {
this.identifier = id;
}
//
// Cached state store data
//
/**
* The last time the state store cache was fully updated.
*
* @return Timestamp.
*/
public long getCacheUpdateTime() {
return this.cacheLastUpdateTime;
}
/**
* Stops the cache update service.
*/
@VisibleForTesting
public void stopCacheUpdateService() {
if (this.cacheUpdater != null) {
this.cacheUpdater.stop();
removeService(this.cacheUpdater);
this.cacheUpdater = null;
}
}
/**
* Register a cached record store for automatic periodic cache updates.
*
* @param client Client to the state store.
*/
public void registerCacheExternal(StateStoreCache client) {
this.cachesToUpdateExternal.add(client);
}
/**
* Refresh the cache with information from the State Store. Called
* periodically by the CacheUpdateService to maintain data caches and
* versions.
*/
public void refreshCaches() {
refreshCaches(false);
}
/**
* Refresh the cache with information from the State Store. Called
* periodically by the CacheUpdateService to maintain data caches and
* versions.
* @param force If we force the refresh.
*/
public void refreshCaches(boolean force) {
boolean success = true;
if (isDriverReady()) {
List<StateStoreCache> cachesToUpdate = new LinkedList<>();
cachesToUpdate.addAll(cachesToUpdateInternal);
cachesToUpdate.addAll(cachesToUpdateExternal);
for (StateStoreCache cachedStore : cachesToUpdate) {
String cacheName = cachedStore.getClass().getSimpleName();
boolean result = false;
try {
result = cachedStore.loadCache(force);
} catch (IOException e) {
LOG.error("Error updating cache for {}", cacheName, e);
result = false;
}
if (!result) {
success = false;
LOG.error("Cache update failed for cache {}", cacheName);
}
}
} else {
success = false;
LOG.info("Skipping State Store cache update, driver is not ready.");
}
if (success) {
// Uses local time, not driver time.
this.cacheLastUpdateTime = Time.now();
}
}
/**
* Update the cache for a specific record store.
*
* @param clazz Class of the record store.
* @return If the cached was loaded.
* @throws IOException if the cache update failed.
*/
public boolean loadCache(final Class<?> clazz) throws IOException {
return loadCache(clazz, false);
}
/**
* Update the cache for a specific record store.
*
* @param clazz Class of the record store.
* @param force Force the update ignoring cached periods.
* @return If the cached was loaded.
* @throws IOException if the cache update failed.
*/
public boolean loadCache(Class<?> clazz, boolean force) throws IOException {
List<StateStoreCache> cachesToUpdate =
new LinkedList<StateStoreCache>();
cachesToUpdate.addAll(this.cachesToUpdateInternal);
cachesToUpdate.addAll(this.cachesToUpdateExternal);
for (StateStoreCache cachedStore : cachesToUpdate) {
if (clazz.isInstance(cachedStore)) {
return cachedStore.loadCache(force);
}
}
throw new IOException("Registered cache was not found for " + clazz);
}
}

View File

@ -0,0 +1,311 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.impl;
import static org.apache.hadoop.hdfs.server.federation.store.StateStoreUtils.filterMultiple;
import java.io.IOException;
import java.util.ArrayList;
import java.util.Collection;
import java.util.Collections;
import java.util.HashMap;
import java.util.HashSet;
import java.util.LinkedList;
import java.util.List;
import java.util.Map;
import java.util.Set;
import java.util.TreeSet;
import java.util.concurrent.locks.Lock;
import java.util.concurrent.locks.ReadWriteLock;
import java.util.concurrent.locks.ReentrantReadWriteLock;
import org.apache.hadoop.classification.InterfaceAudience;
import org.apache.hadoop.classification.InterfaceStability;
import org.apache.hadoop.hdfs.server.federation.resolver.FederationNamenodeServiceState;
import org.apache.hadoop.hdfs.server.federation.resolver.FederationNamespaceInfo;
import org.apache.hadoop.hdfs.server.federation.store.MembershipStore;
import org.apache.hadoop.hdfs.server.federation.store.StateStoreCache;
import org.apache.hadoop.hdfs.server.federation.store.driver.StateStoreDriver;
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetNamenodeRegistrationsRequest;
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetNamenodeRegistrationsResponse;
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetNamespaceInfoRequest;
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetNamespaceInfoResponse;
import org.apache.hadoop.hdfs.server.federation.store.protocol.NamenodeHeartbeatRequest;
import org.apache.hadoop.hdfs.server.federation.store.protocol.NamenodeHeartbeatResponse;
import org.apache.hadoop.hdfs.server.federation.store.protocol.UpdateNamenodeRegistrationRequest;
import org.apache.hadoop.hdfs.server.federation.store.protocol.UpdateNamenodeRegistrationResponse;
import org.apache.hadoop.hdfs.server.federation.store.records.MembershipState;
import org.apache.hadoop.hdfs.server.federation.store.records.Query;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
/**
* Implementation of the {@link MembershipStore} State Store API.
*/
@InterfaceAudience.Private
@InterfaceStability.Evolving
public class MembershipStoreImpl
extends MembershipStore implements StateStoreCache {
private static final Logger LOG =
LoggerFactory.getLogger(MembershipStoreImpl.class);
/** Reported namespaces that are not decommissioned. */
private final Set<FederationNamespaceInfo> activeNamespaces;
/** Namenodes (after evaluating the quorum) that are active in the cluster. */
private final Map<String, MembershipState> activeRegistrations;
/** Namenode status reports (raw) that were discarded for being too old. */
private final Map<String, MembershipState> expiredRegistrations;
/** Lock to access the local memory cache. */
private final ReadWriteLock cacheReadWriteLock =
new ReentrantReadWriteLock();
private final Lock cacheReadLock = cacheReadWriteLock.readLock();
private final Lock cacheWriteLock = cacheReadWriteLock.writeLock();
public MembershipStoreImpl(StateStoreDriver driver) {
super(driver);
this.activeRegistrations = new HashMap<>();
this.expiredRegistrations = new HashMap<>();
this.activeNamespaces = new TreeSet<>();
}
@Override
public GetNamenodeRegistrationsResponse getExpiredNamenodeRegistrations(
GetNamenodeRegistrationsRequest request) throws IOException {
GetNamenodeRegistrationsResponse response =
GetNamenodeRegistrationsResponse.newInstance();
cacheReadLock.lock();
try {
Collection<MembershipState> vals = this.expiredRegistrations.values();
List<MembershipState> copyVals = new ArrayList<>(vals);
response.setNamenodeMemberships(copyVals);
} finally {
cacheReadLock.unlock();
}
return response;
}
@Override
public GetNamespaceInfoResponse getNamespaceInfo(
GetNamespaceInfoRequest request) throws IOException {
Set<FederationNamespaceInfo> namespaces = new HashSet<>();
try {
cacheReadLock.lock();
namespaces.addAll(activeNamespaces);
} finally {
cacheReadLock.unlock();
}
GetNamespaceInfoResponse response =
GetNamespaceInfoResponse.newInstance(namespaces);
return response;
}
@Override
public GetNamenodeRegistrationsResponse getNamenodeRegistrations(
final GetNamenodeRegistrationsRequest request) throws IOException {
// TODO Cache some common queries and sorts
List<MembershipState> ret = null;
cacheReadLock.lock();
try {
Collection<MembershipState> registrations = activeRegistrations.values();
MembershipState partialMembership = request.getPartialMembership();
if (partialMembership == null) {
ret = new ArrayList<>(registrations);
} else {
Query<MembershipState> query = new Query<>(partialMembership);
ret = filterMultiple(query, registrations);
}
} finally {
cacheReadLock.unlock();
}
// Sort in ascending update date order
Collections.sort(ret);
GetNamenodeRegistrationsResponse response =
GetNamenodeRegistrationsResponse.newInstance(ret);
return response;
}
@Override
public NamenodeHeartbeatResponse namenodeHeartbeat(
NamenodeHeartbeatRequest request) throws IOException {
MembershipState record = request.getNamenodeMembership();
String nnId = record.getNamenodeKey();
MembershipState existingEntry = null;
cacheReadLock.lock();
try {
existingEntry = this.activeRegistrations.get(nnId);
} finally {
cacheReadLock.unlock();
}
if (existingEntry != null) {
if (existingEntry.getState() != record.getState()) {
LOG.info("NN registration state has changed: {} -> {}",
existingEntry, record);
} else {
LOG.debug("Updating NN registration: {} -> {}", existingEntry, record);
}
} else {
LOG.info("Inserting new NN registration: {}", record);
}
boolean status = getDriver().put(record, true, false);
NamenodeHeartbeatResponse response =
NamenodeHeartbeatResponse.newInstance(status);
return response;
}
@Override
public boolean loadCache(boolean force) throws IOException {
super.loadCache(force);
// Update local cache atomically
cacheWriteLock.lock();
try {
this.activeRegistrations.clear();
this.expiredRegistrations.clear();
this.activeNamespaces.clear();
// Build list of NN registrations: nnId -> registration list
Map<String, List<MembershipState>> nnRegistrations = new HashMap<>();
List<MembershipState> cachedRecords = getCachedRecords();
for (MembershipState membership : cachedRecords) {
String nnId = membership.getNamenodeKey();
if (membership.getState() == FederationNamenodeServiceState.EXPIRED) {
// Expired, RPC service does not use these
String key = membership.getPrimaryKey();
this.expiredRegistrations.put(key, membership);
} else {
// This is a valid NN registration, build a list of all registrations
// using the NN id to use for the quorum calculation.
List<MembershipState> nnRegistration =
nnRegistrations.get(nnId);
if (nnRegistration == null) {
nnRegistration = new LinkedList<>();
nnRegistrations.put(nnId, nnRegistration);
}
nnRegistration.add(membership);
String bpId = membership.getBlockPoolId();
String cId = membership.getClusterId();
String nsId = membership.getNameserviceId();
FederationNamespaceInfo nsInfo =
new FederationNamespaceInfo(bpId, cId, nsId);
this.activeNamespaces.add(nsInfo);
}
}
// Calculate most representative entry for each active NN id
for (List<MembershipState> nnRegistration : nnRegistrations.values()) {
// Run quorum based on NN state
MembershipState representativeRecord =
getRepresentativeQuorum(nnRegistration);
String nnKey = representativeRecord.getNamenodeKey();
this.activeRegistrations.put(nnKey, representativeRecord);
}
LOG.debug("Refreshed {} NN registrations from State Store",
cachedRecords.size());
} finally {
cacheWriteLock.unlock();
}
return true;
}
@Override
public UpdateNamenodeRegistrationResponse updateNamenodeRegistration(
UpdateNamenodeRegistrationRequest request) throws IOException {
boolean status = false;
cacheWriteLock.lock();
try {
String namenode = MembershipState.getNamenodeKey(
request.getNameserviceId(), request.getNamenodeId());
MembershipState member = this.activeRegistrations.get(namenode);
if (member != null) {
member.setState(request.getState());
status = true;
}
} finally {
cacheWriteLock.unlock();
}
UpdateNamenodeRegistrationResponse response =
UpdateNamenodeRegistrationResponse.newInstance(status);
return response;
}
/**
* Picks the most recent entry in the subset that is most agreeable on the
* specified field. 1) If a majority of the collection has the same value for
* the field, the first sorted entry within the subset the matches the
* majority value 2) Otherwise the first sorted entry in the set of all
* entries
*
* @param entries - Collection of state store record objects of the same type
* @param fieldName - Field name for the value to compare
* @return record that is most representative of the field name
*/
private MembershipState getRepresentativeQuorum(
Collection<MembershipState> records) {
// Collate objects by field value: field value -> order set of records
Map<FederationNamenodeServiceState, TreeSet<MembershipState>> occurenceMap =
new HashMap<>();
for (MembershipState record : records) {
FederationNamenodeServiceState state = record.getState();
TreeSet<MembershipState> matchingSet = occurenceMap.get(state);
if (matchingSet == null) {
// TreeSet orders elements by descending date via comparators
matchingSet = new TreeSet<>();
occurenceMap.put(state, matchingSet);
}
matchingSet.add(record);
}
// Select largest group
TreeSet<MembershipState> largestSet = new TreeSet<>();
for (TreeSet<MembershipState> matchingSet : occurenceMap.values()) {
if (largestSet.size() < matchingSet.size()) {
largestSet = matchingSet;
}
}
// If quorum, use the newest element here
if (largestSet.size() > records.size() / 2) {
return largestSet.first();
// Otherwise, return most recent by class comparator
} else if (records.size() > 0) {
TreeSet<MembershipState> sortedList = new TreeSet<>(records);
LOG.debug("Quorum failed, using most recent: {}", sortedList.first());
return sortedList.first();
} else {
return null;
}
}
}

View File

@ -0,0 +1,31 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
/**
* Contains implementations of the state store API interfaces. All classes
* derive from {@link
* org.apache.hadoop.hdfs.server.federation.store.RecordStore}. The API
* definitions are contained in the
* org.apache.hadoop.hdfs.server.federation.store package.
*/
@InterfaceAudience.Private
@InterfaceStability.Evolving
package org.apache.hadoop.hdfs.server.federation.store.impl;
import org.apache.hadoop.classification.InterfaceAudience;
import org.apache.hadoop.classification.InterfaceStability;

View File

@ -0,0 +1,52 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.protocol;
import java.io.IOException;
import org.apache.hadoop.classification.InterfaceAudience.Public;
import org.apache.hadoop.classification.InterfaceStability.Unstable;
import org.apache.hadoop.hdfs.server.federation.store.driver.StateStoreSerializer;
import org.apache.hadoop.hdfs.server.federation.store.records.MembershipState;
/**
* API request for listing namenode registrations present in the state store.
*/
public abstract class GetNamenodeRegistrationsRequest {
public static GetNamenodeRegistrationsRequest newInstance()
throws IOException {
return StateStoreSerializer.newRecord(
GetNamenodeRegistrationsRequest.class);
}
public static GetNamenodeRegistrationsRequest newInstance(
MembershipState member) throws IOException {
GetNamenodeRegistrationsRequest request = newInstance();
request.setPartialMembership(member);
return request;
}
@Public
@Unstable
public abstract MembershipState getPartialMembership();
@Public
@Unstable
public abstract void setPartialMembership(MembershipState member);
}

View File

@ -0,0 +1,55 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.protocol;
import java.io.IOException;
import java.util.List;
import org.apache.hadoop.classification.InterfaceAudience.Public;
import org.apache.hadoop.classification.InterfaceStability.Unstable;
import org.apache.hadoop.hdfs.server.federation.store.driver.StateStoreSerializer;
import org.apache.hadoop.hdfs.server.federation.store.records.MembershipState;
/**
* API response for listing namenode registrations present in the state store.
*/
public abstract class GetNamenodeRegistrationsResponse {
public static GetNamenodeRegistrationsResponse newInstance()
throws IOException {
return StateStoreSerializer.newRecord(
GetNamenodeRegistrationsResponse.class);
}
public static GetNamenodeRegistrationsResponse newInstance(
List<MembershipState> records) throws IOException {
GetNamenodeRegistrationsResponse response = newInstance();
response.setNamenodeMemberships(records);
return response;
}
@Public
@Unstable
public abstract List<MembershipState> getNamenodeMemberships()
throws IOException;
@Public
@Unstable
public abstract void setNamenodeMemberships(
List<MembershipState> records) throws IOException;
}

View File

@ -0,0 +1,30 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.protocol;
import org.apache.hadoop.hdfs.server.federation.store.driver.StateStoreSerializer;
/**
* API response for listing HDFS namespaces present in the state store.
*/
public abstract class GetNamespaceInfoRequest {
public static GetNamespaceInfoRequest newInstance() {
return StateStoreSerializer.newRecord(GetNamespaceInfoRequest.class);
}
}

View File

@ -0,0 +1,52 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.protocol;
import java.io.IOException;
import java.util.Set;
import org.apache.hadoop.classification.InterfaceAudience.Public;
import org.apache.hadoop.classification.InterfaceStability.Unstable;
import org.apache.hadoop.hdfs.server.federation.resolver.FederationNamespaceInfo;
import org.apache.hadoop.hdfs.server.federation.store.driver.StateStoreSerializer;
/**
* API response for listing HDFS namespaces present in the state store.
*/
public abstract class GetNamespaceInfoResponse {
public static GetNamespaceInfoResponse newInstance() {
return StateStoreSerializer.newRecord(GetNamespaceInfoResponse.class);
}
public static GetNamespaceInfoResponse newInstance(
Set<FederationNamespaceInfo> namespaces) throws IOException {
GetNamespaceInfoResponse response = newInstance();
response.setNamespaceInfo(namespaces);
return response;
}
@Public
@Unstable
public abstract Set<FederationNamespaceInfo> getNamespaceInfo();
@Public
@Unstable
public abstract void setNamespaceInfo(
Set<FederationNamespaceInfo> namespaceInfo);
}

View File

@ -0,0 +1,52 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.protocol;
import java.io.IOException;
import org.apache.hadoop.classification.InterfaceAudience.Private;
import org.apache.hadoop.classification.InterfaceStability.Unstable;
import org.apache.hadoop.hdfs.server.federation.store.driver.StateStoreSerializer;
import org.apache.hadoop.hdfs.server.federation.store.records.MembershipState;
/**
* API request for registering a namenode with the state store.
*/
public abstract class NamenodeHeartbeatRequest {
public static NamenodeHeartbeatRequest newInstance() throws IOException {
return StateStoreSerializer.newRecord(NamenodeHeartbeatRequest.class);
}
public static NamenodeHeartbeatRequest newInstance(MembershipState namenode)
throws IOException {
NamenodeHeartbeatRequest request = newInstance();
request.setNamenodeMembership(namenode);
return request;
}
@Private
@Unstable
public abstract MembershipState getNamenodeMembership()
throws IOException;
@Private
@Unstable
public abstract void setNamenodeMembership(MembershipState report)
throws IOException;
}

View File

@ -0,0 +1,49 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.protocol;
import java.io.IOException;
import org.apache.hadoop.classification.InterfaceAudience.Private;
import org.apache.hadoop.classification.InterfaceStability.Unstable;
import org.apache.hadoop.hdfs.server.federation.store.driver.StateStoreSerializer;
/**
* API response for registering a namenode with the state store.
*/
public abstract class NamenodeHeartbeatResponse {
public static NamenodeHeartbeatResponse newInstance() throws IOException {
return StateStoreSerializer.newRecord(NamenodeHeartbeatResponse.class);
}
public static NamenodeHeartbeatResponse newInstance(boolean status)
throws IOException {
NamenodeHeartbeatResponse response = newInstance();
response.setResult(status);
return response;
}
@Private
@Unstable
public abstract boolean getResult();
@Private
@Unstable
public abstract void setResult(boolean result);
}

View File

@ -0,0 +1,72 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.protocol;
import java.io.IOException;
import org.apache.hadoop.classification.InterfaceAudience.Private;
import org.apache.hadoop.classification.InterfaceStability.Unstable;
import org.apache.hadoop.hdfs.server.federation.resolver.FederationNamenodeServiceState;
import org.apache.hadoop.hdfs.server.federation.store.driver.StateStoreSerializer;
/**
* API request for overriding an existing namenode registration in the state
* store.
*/
public abstract class UpdateNamenodeRegistrationRequest {
public static UpdateNamenodeRegistrationRequest newInstance()
throws IOException {
return StateStoreSerializer.newRecord(
UpdateNamenodeRegistrationRequest.class);
}
public static UpdateNamenodeRegistrationRequest newInstance(
String nameserviceId, String namenodeId,
FederationNamenodeServiceState state) throws IOException {
UpdateNamenodeRegistrationRequest request = newInstance();
request.setNameserviceId(nameserviceId);
request.setNamenodeId(namenodeId);
request.setState(state);
return request;
}
@Private
@Unstable
public abstract String getNameserviceId();
@Private
@Unstable
public abstract String getNamenodeId();
@Private
@Unstable
public abstract FederationNamenodeServiceState getState();
@Private
@Unstable
public abstract void setNameserviceId(String nsId);
@Private
@Unstable
public abstract void setNamenodeId(String nnId);
@Private
@Unstable
public abstract void setState(FederationNamenodeServiceState state);
}

View File

@ -0,0 +1,51 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.protocol;
import java.io.IOException;
import org.apache.hadoop.classification.InterfaceAudience.Private;
import org.apache.hadoop.classification.InterfaceStability.Unstable;
import org.apache.hadoop.hdfs.server.federation.store.driver.StateStoreSerializer;
/**
* API response for overriding an existing namenode registration in the state
* store.
*/
public abstract class UpdateNamenodeRegistrationResponse {
public static UpdateNamenodeRegistrationResponse newInstance() {
return StateStoreSerializer.newRecord(
UpdateNamenodeRegistrationResponse.class);
}
public static UpdateNamenodeRegistrationResponse newInstance(boolean status)
throws IOException {
UpdateNamenodeRegistrationResponse response = newInstance();
response.setResult(status);
return response;
}
@Private
@Unstable
public abstract boolean getResult();
@Private
@Unstable
public abstract void setResult(boolean result);
}

View File

@ -0,0 +1,145 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.protocol.impl.pb;
import java.io.IOException;
import java.lang.reflect.Method;
import org.apache.commons.codec.binary.Base64;
import com.google.protobuf.GeneratedMessage;
import com.google.protobuf.Message;
import com.google.protobuf.Message.Builder;
import com.google.protobuf.MessageOrBuilder;
/**
* Helper class for setting/getting data elements in an object backed by a
* protobuf implementation.
*/
public class FederationProtocolPBTranslator<P extends GeneratedMessage,
B extends Builder, T extends MessageOrBuilder> {
/** Optional proto byte stream used to create this object. */
private P proto;
/** The class of the proto handler for this translator. */
private Class<P> protoClass;
/** Internal builder, used to store data that has been set. */
private B builder;
public FederationProtocolPBTranslator(Class<P> protoType) {
this.protoClass = protoType;
}
/**
* Called if this translator is to be created from an existing protobuf byte
* stream.
*
* @param p The existing proto object to use to initialize the translator.
* @throws IllegalArgumentException
*/
@SuppressWarnings("unchecked")
public void setProto(Message p) {
if (protoClass.isInstance(p)) {
if (this.builder != null) {
// Merge with builder
this.builder.mergeFrom((P) p);
} else {
// Store proto
this.proto = (P) p;
}
} else {
throw new IllegalArgumentException(
"Cannot decode proto type " + p.getClass().getName());
}
}
/**
* Create or return the cached protobuf builder for this translator.
*
* @return cached Builder instance
*/
@SuppressWarnings("unchecked")
public B getBuilder() {
if (this.builder == null) {
try {
Method method = protoClass.getMethod("newBuilder");
this.builder = (B) method.invoke(null);
if (this.proto != null) {
// Merge in existing immutable proto
this.builder.mergeFrom(this.proto);
}
} catch (ReflectiveOperationException e) {
this.builder = null;
}
}
return this.builder;
}
/**
* Get the serialized proto object. If the translator was created from a byte
* stream, returns the intitial byte stream. Otherwise creates a new byte
* stream from the cached builder.
*
* @return Protobuf message object
*/
@SuppressWarnings("unchecked")
public P build() {
if (this.builder != null) {
// serialize from builder (mutable) first
Message m = this.builder.build();
return (P) m;
} else if (this.proto != null) {
// Use immutable message source, message is unchanged
return this.proto;
}
return null;
}
/**
* Returns an interface to access data stored within this object. The object
* may have been initialized either via a builder or by an existing protobuf
* byte stream.
*
* @return MessageOrBuilder protobuf interface for the requested class.
*/
@SuppressWarnings("unchecked")
public T getProtoOrBuilder() {
if (this.builder != null) {
// Use mutable builder if it exists
return (T) this.builder;
} else if (this.proto != null) {
// Use immutable message source
return (T) this.proto;
} else {
// Construct empty builder
return (T) this.getBuilder();
}
}
/**
* Read instance from base64 data.
* @param base64String
* @throws IOException
*/
@SuppressWarnings("unchecked")
public void readInstance(String base64String) throws IOException {
byte[] bytes = Base64.decodeBase64(base64String);
Message msg = getBuilder().mergeFrom(bytes).build();
this.proto = (P) msg;
}
}

View File

@ -0,0 +1,87 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.protocol.impl.pb;
import java.io.IOException;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.GetNamenodeRegistrationsRequestProto;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.GetNamenodeRegistrationsRequestProtoOrBuilder;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.NamenodeMembershipRecordProto;
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetNamenodeRegistrationsRequest;
import org.apache.hadoop.hdfs.server.federation.store.records.MembershipState;
import org.apache.hadoop.hdfs.server.federation.store.records.impl.pb.MembershipStatePBImpl;
import org.apache.hadoop.hdfs.server.federation.store.records.impl.pb.PBRecord;
import com.google.protobuf.Message;
/**
* Protobuf implementation of the state store API object
* GetNamenodeRegistrationsRequest.
*/
public class GetNamenodeRegistrationsRequestPBImpl
extends GetNamenodeRegistrationsRequest implements PBRecord {
private FederationProtocolPBTranslator<GetNamenodeRegistrationsRequestProto,
GetNamenodeRegistrationsRequestProto.Builder,
GetNamenodeRegistrationsRequestProtoOrBuilder> translator =
new FederationProtocolPBTranslator<
GetNamenodeRegistrationsRequestProto,
GetNamenodeRegistrationsRequestProto.Builder,
GetNamenodeRegistrationsRequestProtoOrBuilder>(
GetNamenodeRegistrationsRequestProto.class);
public GetNamenodeRegistrationsRequestPBImpl() {
}
public GetNamenodeRegistrationsRequestPBImpl(
GetNamenodeRegistrationsRequestProto proto) {
this.translator.setProto(proto);
}
@Override
public GetNamenodeRegistrationsRequestProto getProto() {
return this.translator.build();
}
@Override
public void setProto(Message proto) {
this.translator.setProto(proto);
}
@Override
public void readInstance(String base64String) throws IOException {
this.translator.readInstance(base64String);
}
@Override
public MembershipState getPartialMembership() {
GetNamenodeRegistrationsRequestProtoOrBuilder proto =
this.translator.getProtoOrBuilder();
if (!proto.hasMembership()){
return null;
}
NamenodeMembershipRecordProto memberProto = proto.getMembership();
return new MembershipStatePBImpl(memberProto);
}
@Override
public void setPartialMembership(MembershipState member) {
MembershipStatePBImpl memberPB = (MembershipStatePBImpl)member;
this.translator.getBuilder().setMembership(memberPB.getProto());
}
}

View File

@ -0,0 +1,99 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.protocol.impl.pb;
import java.io.IOException;
import java.util.ArrayList;
import java.util.List;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.GetNamenodeRegistrationsResponseProto;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.GetNamenodeRegistrationsResponseProtoOrBuilder;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.NamenodeMembershipRecordProto;
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetNamenodeRegistrationsResponse;
import org.apache.hadoop.hdfs.server.federation.store.records.MembershipState;
import org.apache.hadoop.hdfs.server.federation.store.records.impl.pb.MembershipStatePBImpl;
import org.apache.hadoop.hdfs.server.federation.store.records.impl.pb.PBRecord;
import com.google.protobuf.Message;
/**
* Protobuf implementation of the state store API object
* GetNamenodeRegistrationsResponse.
*/
public class GetNamenodeRegistrationsResponsePBImpl
extends GetNamenodeRegistrationsResponse implements PBRecord {
private FederationProtocolPBTranslator<GetNamenodeRegistrationsResponseProto,
GetNamenodeRegistrationsResponseProto.Builder,
GetNamenodeRegistrationsResponseProtoOrBuilder> translator =
new FederationProtocolPBTranslator<
GetNamenodeRegistrationsResponseProto,
GetNamenodeRegistrationsResponseProto.Builder,
GetNamenodeRegistrationsResponseProtoOrBuilder>(
GetNamenodeRegistrationsResponseProto.class);
public GetNamenodeRegistrationsResponsePBImpl() {
}
public GetNamenodeRegistrationsResponsePBImpl(
GetNamenodeRegistrationsResponseProto proto) {
this.translator.setProto(proto);
}
@Override
public GetNamenodeRegistrationsResponseProto getProto() {
return this.translator.build();
}
@Override
public void setProto(Message proto) {
this.translator.setProto(proto);
}
@Override
public void readInstance(String base64String) throws IOException {
this.translator.readInstance(base64String);
}
@Override
public List<MembershipState> getNamenodeMemberships()
throws IOException {
List<MembershipState> ret = new ArrayList<MembershipState>();
List<NamenodeMembershipRecordProto> memberships =
this.translator.getProtoOrBuilder().getNamenodeMembershipsList();
for (NamenodeMembershipRecordProto memberProto : memberships) {
MembershipState membership = new MembershipStatePBImpl(memberProto);
ret.add(membership);
}
return ret;
}
@Override
public void setNamenodeMemberships(List<MembershipState> records)
throws IOException {
for (MembershipState member : records) {
if (member instanceof MembershipStatePBImpl) {
MembershipStatePBImpl memberPB = (MembershipStatePBImpl)member;
this.translator.getBuilder().addNamenodeMemberships(
memberPB.getProto());
}
}
}
}

View File

@ -0,0 +1,60 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.protocol.impl.pb;
import java.io.IOException;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.GetNamespaceInfoRequestProto;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.GetNamespaceInfoRequestProto.Builder;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.GetNamespaceInfoRequestProtoOrBuilder;
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetNamespaceInfoRequest;
import org.apache.hadoop.hdfs.server.federation.store.records.impl.pb.PBRecord;
import com.google.protobuf.Message;
/**
* Protobuf implementation of the state store API object
* GetNamespaceInfoRequest.
*/
public class GetNamespaceInfoRequestPBImpl extends GetNamespaceInfoRequest
implements PBRecord {
private FederationProtocolPBTranslator<GetNamespaceInfoRequestProto,
Builder, GetNamespaceInfoRequestProtoOrBuilder> translator =
new FederationProtocolPBTranslator<GetNamespaceInfoRequestProto,
Builder, GetNamespaceInfoRequestProtoOrBuilder>(
GetNamespaceInfoRequestProto.class);
public GetNamespaceInfoRequestPBImpl() {
}
@Override
public GetNamespaceInfoRequestProto getProto() {
return this.translator.build();
}
@Override
public void setProto(Message protocol) {
this.translator.setProto(protocol);
}
@Override
public void readInstance(String base64String) throws IOException {
this.translator.readInstance(base64String);
}
}

View File

@ -0,0 +1,95 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.protocol.impl.pb;
import java.io.IOException;
import java.util.HashSet;
import java.util.List;
import java.util.Set;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.FederationNamespaceInfoProto;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.GetNamespaceInfoResponseProto;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.GetNamespaceInfoResponseProtoOrBuilder;
import org.apache.hadoop.hdfs.server.federation.resolver.FederationNamespaceInfo;
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetNamespaceInfoResponse;
import org.apache.hadoop.hdfs.server.federation.store.records.impl.pb.PBRecord;
import com.google.protobuf.Message;
/**
* Protobuf implementation of the state store API object
* GetNamespaceInfoResponse.
*/
public class GetNamespaceInfoResponsePBImpl
extends GetNamespaceInfoResponse implements PBRecord {
private FederationProtocolPBTranslator<GetNamespaceInfoResponseProto,
GetNamespaceInfoResponseProto.Builder,
GetNamespaceInfoResponseProtoOrBuilder> translator =
new FederationProtocolPBTranslator<GetNamespaceInfoResponseProto,
GetNamespaceInfoResponseProto.Builder,
GetNamespaceInfoResponseProtoOrBuilder>(
GetNamespaceInfoResponseProto.class);
public GetNamespaceInfoResponsePBImpl() {
}
@Override
public GetNamespaceInfoResponseProto getProto() {
return this.translator.build();
}
@Override
public void setProto(Message protocol) {
this.translator.setProto(protocol);
}
@Override
public void readInstance(String base64String) throws IOException {
this.translator.readInstance(base64String);
}
@Override
public Set<FederationNamespaceInfo> getNamespaceInfo() {
Set<FederationNamespaceInfo> ret = new HashSet<FederationNamespaceInfo>();
List<FederationNamespaceInfoProto> namespaceList =
this.translator.getProtoOrBuilder().getNamespaceInfosList();
for (FederationNamespaceInfoProto ns : namespaceList) {
FederationNamespaceInfo info = new FederationNamespaceInfo(
ns.getBlockPoolId(), ns.getClusterId(), ns.getNameserviceId());
ret.add(info);
}
return ret;
}
@Override
public void setNamespaceInfo(Set<FederationNamespaceInfo> namespaceInfo) {
int index = 0;
for (FederationNamespaceInfo item : namespaceInfo) {
FederationNamespaceInfoProto.Builder itemBuilder =
FederationNamespaceInfoProto.newBuilder();
itemBuilder.setClusterId(item.getClusterId());
itemBuilder.setBlockPoolId(item.getBlockPoolId());
itemBuilder.setNameserviceId(item.getNameserviceId());
this.translator.getBuilder().addNamespaceInfos(index,
itemBuilder.build());
index++;
}
}
}

View File

@ -0,0 +1,93 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.protocol.impl.pb;
import java.io.IOException;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.NamenodeHeartbeatRequestProto;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.NamenodeHeartbeatRequestProto.Builder;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.NamenodeHeartbeatRequestProtoOrBuilder;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.NamenodeMembershipRecordProto;
import org.apache.hadoop.hdfs.server.federation.store.driver.StateStoreSerializer;
import org.apache.hadoop.hdfs.server.federation.store.protocol.NamenodeHeartbeatRequest;
import org.apache.hadoop.hdfs.server.federation.store.records.MembershipState;
import org.apache.hadoop.hdfs.server.federation.store.records.impl.pb.MembershipStatePBImpl;
import org.apache.hadoop.hdfs.server.federation.store.records.impl.pb.PBRecord;
import com.google.protobuf.Message;
/**
* Protobuf implementation of the state store API object
* NamenodeHeartbeatRequest.
*/
public class NamenodeHeartbeatRequestPBImpl
extends NamenodeHeartbeatRequest implements PBRecord {
private FederationProtocolPBTranslator<NamenodeHeartbeatRequestProto, Builder,
NamenodeHeartbeatRequestProtoOrBuilder> translator =
new FederationProtocolPBTranslator<NamenodeHeartbeatRequestProto,
Builder,
NamenodeHeartbeatRequestProtoOrBuilder>(
NamenodeHeartbeatRequestProto.class);
public NamenodeHeartbeatRequestPBImpl() {
}
@Override
public NamenodeHeartbeatRequestProto getProto() {
return this.translator.build();
}
@Override
public void setProto(Message proto) {
this.translator.setProto(proto);
}
@Override
public void readInstance(String base64String) throws IOException {
this.translator.readInstance(base64String);
}
@Override
public MembershipState getNamenodeMembership() throws IOException {
NamenodeMembershipRecordProto membershipProto =
this.translator.getProtoOrBuilder().getNamenodeMembership();
MembershipState membership =
StateStoreSerializer.newRecord(MembershipState.class);
if (membership instanceof MembershipStatePBImpl) {
MembershipStatePBImpl membershipPB = (MembershipStatePBImpl)membership;
membershipPB.setProto(membershipProto);
return membershipPB;
} else {
throw new IOException("Cannot get membership from request");
}
}
@Override
public void setNamenodeMembership(MembershipState membership)
throws IOException {
if (membership instanceof MembershipStatePBImpl) {
MembershipStatePBImpl membershipPB = (MembershipStatePBImpl)membership;
NamenodeMembershipRecordProto membershipProto =
(NamenodeMembershipRecordProto)membershipPB.getProto();
this.translator.getBuilder().setNamenodeMembership(membershipProto);
} else {
throw new IOException("Cannot set mount table entry");
}
}
}

View File

@ -0,0 +1,71 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.protocol.impl.pb;
import java.io.IOException;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.NamenodeHeartbeatResponseProto;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.NamenodeHeartbeatResponseProtoOrBuilder;
import org.apache.hadoop.hdfs.server.federation.store.protocol.NamenodeHeartbeatResponse;
import org.apache.hadoop.hdfs.server.federation.store.records.impl.pb.PBRecord;
import com.google.protobuf.Message;
/**
* Protobuf implementation of the state store API object
* NamenodeHeartbeatResponse.
*/
public class NamenodeHeartbeatResponsePBImpl extends NamenodeHeartbeatResponse
implements PBRecord {
private FederationProtocolPBTranslator<NamenodeHeartbeatResponseProto,
NamenodeHeartbeatResponseProto.Builder,
NamenodeHeartbeatResponseProtoOrBuilder> translator =
new FederationProtocolPBTranslator<NamenodeHeartbeatResponseProto,
NamenodeHeartbeatResponseProto.Builder,
NamenodeHeartbeatResponseProtoOrBuilder>(
NamenodeHeartbeatResponseProto.class);
public NamenodeHeartbeatResponsePBImpl() {
}
@Override
public NamenodeHeartbeatResponseProto getProto() {
return this.translator.build();
}
@Override
public void setProto(Message proto) {
this.translator.setProto(proto);
}
@Override
public void readInstance(String base64String) throws IOException {
this.translator.readInstance(base64String);
}
@Override
public boolean getResult() {
return this.translator.getProtoOrBuilder().getStatus();
}
@Override
public void setResult(boolean result) {
this.translator.getBuilder().setStatus(result);
}
}

View File

@ -0,0 +1,95 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.protocol.impl.pb;
import java.io.IOException;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.UpdateNamenodeRegistrationRequestProto;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.UpdateNamenodeRegistrationRequestProtoOrBuilder;
import org.apache.hadoop.hdfs.server.federation.resolver.FederationNamenodeServiceState;
import org.apache.hadoop.hdfs.server.federation.store.protocol.UpdateNamenodeRegistrationRequest;
import org.apache.hadoop.hdfs.server.federation.store.records.impl.pb.PBRecord;
import com.google.protobuf.Message;
/**
* Protobuf implementation of the state store API object
* OverrideNamenodeRegistrationRequest.
*/
public class UpdateNamenodeRegistrationRequestPBImpl
extends UpdateNamenodeRegistrationRequest implements PBRecord {
private FederationProtocolPBTranslator<
UpdateNamenodeRegistrationRequestProto,
UpdateNamenodeRegistrationRequestProto.Builder,
UpdateNamenodeRegistrationRequestProtoOrBuilder> translator =
new FederationProtocolPBTranslator<
UpdateNamenodeRegistrationRequestProto,
UpdateNamenodeRegistrationRequestProto.Builder,
UpdateNamenodeRegistrationRequestProtoOrBuilder>(
UpdateNamenodeRegistrationRequestProto.class);
public UpdateNamenodeRegistrationRequestPBImpl() {
}
@Override
public UpdateNamenodeRegistrationRequestProto getProto() {
return this.translator.build();
}
@Override
public void setProto(Message protocol) {
this.translator.setProto(protocol);
}
@Override
public void readInstance(String base64String) throws IOException {
this.translator.readInstance(base64String);
}
@Override
public String getNameserviceId() {
return this.translator.getProtoOrBuilder().getNameserviceId();
}
@Override
public String getNamenodeId() {
return this.translator.getProtoOrBuilder().getNamenodeId();
}
@Override
public FederationNamenodeServiceState getState() {
return FederationNamenodeServiceState
.valueOf(this.translator.getProtoOrBuilder().getState());
}
@Override
public void setNameserviceId(String nsId) {
this.translator.getBuilder().setNameserviceId(nsId);
}
@Override
public void setNamenodeId(String nnId) {
this.translator.getBuilder().setNamenodeId(nnId);
}
@Override
public void setState(FederationNamenodeServiceState state) {
this.translator.getBuilder().setState(state.toString());
}
}

View File

@ -0,0 +1,73 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.protocol.impl.pb;
import java.io.IOException;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.UpdateNamenodeRegistrationResponseProto;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.UpdateNamenodeRegistrationResponseProtoOrBuilder;
import org.apache.hadoop.hdfs.server.federation.store.protocol.UpdateNamenodeRegistrationResponse;
import org.apache.hadoop.hdfs.server.federation.store.records.impl.pb.PBRecord;
import com.google.protobuf.Message;
/**
* Protobuf implementation of the state store API object
* OverrideNamenodeRegistrationResponse.
*/
public class UpdateNamenodeRegistrationResponsePBImpl
extends UpdateNamenodeRegistrationResponse implements PBRecord {
private FederationProtocolPBTranslator<
UpdateNamenodeRegistrationResponseProto,
UpdateNamenodeRegistrationResponseProto.Builder,
UpdateNamenodeRegistrationResponseProtoOrBuilder> translator =
new FederationProtocolPBTranslator<
UpdateNamenodeRegistrationResponseProto,
UpdateNamenodeRegistrationResponseProto.Builder,
UpdateNamenodeRegistrationResponseProtoOrBuilder>(
UpdateNamenodeRegistrationResponseProto.class);
public UpdateNamenodeRegistrationResponsePBImpl() {
}
@Override
public UpdateNamenodeRegistrationResponseProto getProto() {
return this.translator.build();
}
@Override
public void setProto(Message proto) {
this.translator.setProto(proto);
}
@Override
public void readInstance(String base64String) throws IOException {
this.translator.readInstance(base64String);
}
@Override
public boolean getResult() {
return this.translator.getProtoOrBuilder().getStatus();
}
@Override
public void setResult(boolean result) {
this.translator.getBuilder().setStatus(result);
}
}

View File

@ -0,0 +1,29 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
/**
* Protobuf implementations of FederationProtocolBase request/response objects
* used by state store APIs. Each state store API is defined in the
* org.apache.hadoop.hdfs.server.federation.store.protocol package.
*/
@InterfaceAudience.Private
@InterfaceStability.Evolving
package org.apache.hadoop.hdfs.server.federation.store.protocol.impl.pb;
import org.apache.hadoop.classification.InterfaceAudience;
import org.apache.hadoop.classification.InterfaceStability;

View File

@ -0,0 +1,329 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.records;
import static org.apache.hadoop.hdfs.server.federation.resolver.FederationNamenodeServiceState.ACTIVE;
import static org.apache.hadoop.hdfs.server.federation.resolver.FederationNamenodeServiceState.EXPIRED;
import static org.apache.hadoop.hdfs.server.federation.resolver.FederationNamenodeServiceState.UNAVAILABLE;
import java.io.IOException;
import java.util.Comparator;
import java.util.SortedMap;
import java.util.TreeMap;
import org.apache.hadoop.hdfs.server.federation.resolver.FederationNamenodeContext;
import org.apache.hadoop.hdfs.server.federation.resolver.FederationNamenodeServiceState;
import org.apache.hadoop.hdfs.server.federation.store.driver.StateStoreSerializer;
/**
* Data schema for storing NN registration information in the
* {@link org.apache.hadoop.hdfs.server.federation.store.StateStoreService
* FederationStateStoreService}.
*/
public abstract class MembershipState extends BaseRecord
implements FederationNamenodeContext {
/** Expiration time in ms for this entry. */
private static long expirationMs;
/** Comparator based on the name.*/
public static final Comparator<MembershipState> NAME_COMPARATOR =
new Comparator<MembershipState>() {
public int compare(MembershipState m1, MembershipState m2) {
return m1.compareNameTo(m2);
}
};
/**
* Constructors.
*/
public MembershipState() {
super();
}
/**
* Create a new membership instance.
* @return Membership instance.
* @throws IOException
*/
public static MembershipState newInstance() {
MembershipState record =
StateStoreSerializer.newRecord(MembershipState.class);
record.init();
return record;
}
/**
* Create a new membership instance.
*
* @param router Identifier of the router.
* @param nameservice Identifier of the nameservice.
* @param namenode Identifier of the namenode.
* @param clusterId Identifier of the cluster.
* @param blockPoolId Identifier of the blockpool.
* @param rpcAddress RPC address.
* @param serviceAddress Service RPC address.
* @param lifelineAddress Lifeline RPC address.
* @param webAddress HTTP address.
* @param state State of the federation.
* @param safemode If the safe mode is enabled.
* @return Membership instance.
* @throws IOException If we cannot create the instance.
*/
public static MembershipState newInstance(String router, String nameservice,
String namenode, String clusterId, String blockPoolId, String rpcAddress,
String serviceAddress, String lifelineAddress, String webAddress,
FederationNamenodeServiceState state, boolean safemode) {
MembershipState record = MembershipState.newInstance();
record.setRouterId(router);
record.setNameserviceId(nameservice);
record.setNamenodeId(namenode);
record.setRpcAddress(rpcAddress);
record.setServiceAddress(serviceAddress);
record.setLifelineAddress(lifelineAddress);
record.setWebAddress(webAddress);
record.setIsSafeMode(safemode);
record.setState(state);
record.setClusterId(clusterId);
record.setBlockPoolId(blockPoolId);
record.validate();
return record;
}
public abstract void setRouterId(String routerId);
public abstract String getRouterId();
public abstract void setNameserviceId(String nameserviceId);
public abstract void setNamenodeId(String namenodeId);
public abstract void setWebAddress(String webAddress);
public abstract void setRpcAddress(String rpcAddress);
public abstract void setServiceAddress(String serviceAddress);
public abstract void setLifelineAddress(String lifelineAddress);
public abstract void setIsSafeMode(boolean isSafeMode);
public abstract void setClusterId(String clusterId);
public abstract void setBlockPoolId(String blockPoolId);
public abstract void setState(FederationNamenodeServiceState state);
public abstract String getNameserviceId();
public abstract String getNamenodeId();
public abstract String getClusterId();
public abstract String getBlockPoolId();
public abstract String getRpcAddress();
public abstract String getServiceAddress();
public abstract String getLifelineAddress();
public abstract String getWebAddress();
public abstract boolean getIsSafeMode();
public abstract FederationNamenodeServiceState getState();
public abstract void setStats(MembershipStats stats);
public abstract MembershipStats getStats() throws IOException;
public abstract void setLastContact(long contact);
public abstract long getLastContact();
@Override
public boolean like(BaseRecord o) {
if (o instanceof MembershipState) {
MembershipState other = (MembershipState)o;
if (getRouterId() != null &&
!getRouterId().equals(other.getRouterId())) {
return false;
}
if (getNameserviceId() != null &&
!getNameserviceId().equals(other.getNameserviceId())) {
return false;
}
if (getNamenodeId() != null &&
!getNamenodeId().equals(other.getNamenodeId())) {
return false;
}
if (getRpcAddress() != null &&
!getRpcAddress().equals(other.getRpcAddress())) {
return false;
}
if (getClusterId() != null &&
!getClusterId().equals(other.getClusterId())) {
return false;
}
if (getBlockPoolId() != null &&
!getBlockPoolId().equals(other.getBlockPoolId())) {
return false;
}
if (getState() != null &&
!getState().equals(other.getState())) {
return false;
}
return true;
}
return false;
}
@Override
public String toString() {
return getRouterId() + "->" + getNameserviceId() + ":" + getNamenodeId()
+ ":" + getRpcAddress() + "-" + getState();
}
@Override
public SortedMap<String, String> getPrimaryKeys() {
SortedMap<String, String> map = new TreeMap<String, String>();
map.put("routerId", getRouterId());
map.put("nameserviceId", getNameserviceId());
map.put("namenodeId", getNamenodeId());
return map;
}
/**
* Check if the namenode is available.
*
* @return If the namenode is available.
*/
public boolean isAvailable() {
return getState() == ACTIVE;
}
/**
* Validates the entry. Throws an IllegalArgementException if the data record
* is missing required information.
*/
@Override
public boolean validate() {
boolean ret = super.validate();
if (getNameserviceId() == null || getNameserviceId().length() == 0) {
//LOG.error("Invalid registration, no nameservice specified " + this);
ret = false;
}
if (getWebAddress() == null || getWebAddress().length() == 0) {
//LOG.error("Invalid registration, no web address specified " + this);
ret = false;
}
if (getRpcAddress() == null || getRpcAddress().length() == 0) {
//LOG.error("Invalid registration, no rpc address specified " + this);
ret = false;
}
if (!isBadState() &&
(getBlockPoolId().isEmpty() || getBlockPoolId().length() == 0)) {
//LOG.error("Invalid registration, no block pool specified " + this);
ret = false;
}
return ret;
}
/**
* Overrides the cached getBlockPoolId() with an update. The state will be
* reset when the cache is flushed
*
* @param newState Service state of the namenode.
*/
public void overrideState(FederationNamenodeServiceState newState) {
this.setState(newState);
}
/**
* Sort by nameservice, namenode, and router.
*
* @param other Another membership to compare to.
* @return If this object goes before the parameter.
*/
public int compareNameTo(MembershipState other) {
int ret = this.getNameserviceId().compareTo(other.getNameserviceId());
if (ret == 0) {
ret = this.getNamenodeId().compareTo(other.getNamenodeId());
}
if (ret == 0) {
ret = this.getRouterId().compareTo(other.getRouterId());
}
return ret;
}
/**
* Get the identifier of this namenode registration.
* @return Identifier of the namenode.
*/
public String getNamenodeKey() {
return getNamenodeKey(this.getNameserviceId(), this.getNamenodeId());
}
/**
* Generate the identifier for a Namenode in the HDFS federation.
*
* @param nsId Nameservice of the Namenode.
* @param nnId Namenode within the Nameservice (HA).
* @return Namenode identifier within the federation.
*/
public static String getNamenodeKey(String nsId, String nnId) {
return nsId + "-" + nnId;
}
/**
* Check if the membership is in a bad state (expired or unavailable).
* @return If the membership is in a bad state (expired or unavailable).
*/
private boolean isBadState() {
return this.getState() == EXPIRED || this.getState() == UNAVAILABLE;
}
@Override
public boolean checkExpired(long currentTime) {
if (super.checkExpired(currentTime)) {
this.setState(EXPIRED);
// Commit it
return true;
}
return false;
}
@Override
public long getExpirationMs() {
return MembershipState.expirationMs;
}
/**
* Set the expiration time for this class.
*
* @param time Expiration time in milliseconds.
*/
public static void setExpirationMs(long time) {
MembershipState.expirationMs = time;
}
}

View File

@ -0,0 +1,126 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.records;
import java.io.IOException;
import java.util.SortedMap;
import java.util.TreeMap;
import org.apache.hadoop.hdfs.server.federation.store.driver.StateStoreSerializer;
/**
* Data schema for storing NN stats in the
* {@link org.apache.hadoop.hdfs.server.federation.store.StateStoreService
* StateStoreService}.
*/
public abstract class MembershipStats extends BaseRecord {
public static MembershipStats newInstance() throws IOException {
MembershipStats record =
StateStoreSerializer.newRecord(MembershipStats.class);
record.init();
return record;
}
public abstract void setTotalSpace(long space);
public abstract long getTotalSpace();
public abstract void setAvailableSpace(long space);
public abstract long getAvailableSpace();
public abstract void setNumOfFiles(long files);
public abstract long getNumOfFiles();
public abstract void setNumOfBlocks(long blocks);
public abstract long getNumOfBlocks();
public abstract void setNumOfBlocksMissing(long blocks);
public abstract long getNumOfBlocksMissing();
public abstract void setNumOfBlocksPendingReplication(long blocks);
public abstract long getNumOfBlocksPendingReplication();
public abstract void setNumOfBlocksUnderReplicated(long blocks);
public abstract long getNumOfBlocksUnderReplicated();
public abstract void setNumOfBlocksPendingDeletion(long blocks);
public abstract long getNumOfBlocksPendingDeletion();
public abstract void setNumOfActiveDatanodes(int nodes);
public abstract int getNumOfActiveDatanodes();
public abstract void setNumOfDeadDatanodes(int nodes);
public abstract int getNumOfDeadDatanodes();
public abstract void setNumOfDecommissioningDatanodes(int nodes);
public abstract int getNumOfDecommissioningDatanodes();
public abstract void setNumOfDecomActiveDatanodes(int nodes);
public abstract int getNumOfDecomActiveDatanodes();
public abstract void setNumOfDecomDeadDatanodes(int nodes);
public abstract int getNumOfDecomDeadDatanodes();
@Override
public SortedMap<String, String> getPrimaryKeys() {
// This record is not stored directly, no key needed
SortedMap<String, String> map = new TreeMap<String, String>();
return map;
}
@Override
public long getExpirationMs() {
// This record is not stored directly, no expiration needed
return -1;
}
@Override
public void setDateModified(long time) {
// We don't store this record directly
}
@Override
public long getDateModified() {
// We don't store this record directly
return 0;
}
@Override
public void setDateCreated(long time) {
// We don't store this record directly
}
@Override
public long getDateCreated() {
// We don't store this record directly
return 0;
}
}

View File

@ -0,0 +1,334 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.records.impl.pb;
import java.io.IOException;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.NamenodeMembershipRecordProto;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.NamenodeMembershipRecordProto.Builder;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.NamenodeMembershipRecordProtoOrBuilder;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.NamenodeMembershipStatsRecordProto;
import org.apache.hadoop.hdfs.server.federation.resolver.FederationNamenodeServiceState;
import org.apache.hadoop.hdfs.server.federation.store.driver.StateStoreSerializer;
import org.apache.hadoop.hdfs.server.federation.store.protocol.impl.pb.FederationProtocolPBTranslator;
import org.apache.hadoop.hdfs.server.federation.store.records.MembershipState;
import org.apache.hadoop.hdfs.server.federation.store.records.MembershipStats;
import com.google.protobuf.Message;
/**
* Protobuf implementation of the MembershipState record.
*/
public class MembershipStatePBImpl extends MembershipState implements PBRecord {
private FederationProtocolPBTranslator<NamenodeMembershipRecordProto, Builder,
NamenodeMembershipRecordProtoOrBuilder> translator =
new FederationProtocolPBTranslator<NamenodeMembershipRecordProto,
Builder, NamenodeMembershipRecordProtoOrBuilder>(
NamenodeMembershipRecordProto.class);
public MembershipStatePBImpl() {
}
public MembershipStatePBImpl(NamenodeMembershipRecordProto proto) {
this.translator.setProto(proto);
}
@Override
public NamenodeMembershipRecordProto getProto() {
return this.translator.build();
}
@Override
public void setProto(Message proto) {
this.translator.setProto(proto);
}
@Override
public void readInstance(String base64String) throws IOException {
this.translator.readInstance(base64String);
}
@Override
public void setRouterId(String routerId) {
Builder builder = this.translator.getBuilder();
if (routerId == null) {
builder.clearRouterId();
} else {
builder.setRouterId(routerId);
}
}
@Override
public void setNameserviceId(String nameserviceId) {
Builder builder = this.translator.getBuilder();
if (nameserviceId == null) {
builder.clearNameserviceId();
} else {
builder.setNameserviceId(nameserviceId);
}
}
@Override
public void setNamenodeId(String namenodeId) {
Builder builder = this.translator.getBuilder();
if (namenodeId == null) {
builder.clearNamenodeId();
} else {
builder.setNamenodeId(namenodeId);
}
}
@Override
public void setWebAddress(String webAddress) {
Builder builder = this.translator.getBuilder();
if (webAddress == null) {
builder.clearWebAddress();
} else {
builder.setWebAddress(webAddress);
}
}
@Override
public void setRpcAddress(String rpcAddress) {
Builder builder = this.translator.getBuilder();
if (rpcAddress == null) {
builder.clearRpcAddress();
} else {
builder.setRpcAddress(rpcAddress);
}
}
@Override
public void setServiceAddress(String serviceAddress) {
this.translator.getBuilder().setServiceAddress(serviceAddress);
}
@Override
public void setLifelineAddress(String lifelineAddress) {
Builder builder = this.translator.getBuilder();
if (lifelineAddress == null) {
builder.clearLifelineAddress();
} else {
builder.setLifelineAddress(lifelineAddress);
}
}
@Override
public void setIsSafeMode(boolean isSafeMode) {
Builder builder = this.translator.getBuilder();
builder.setIsSafeMode(isSafeMode);
}
@Override
public void setClusterId(String clusterId) {
Builder builder = this.translator.getBuilder();
if (clusterId == null) {
builder.clearClusterId();
} else {
builder.setClusterId(clusterId);
}
}
@Override
public void setBlockPoolId(String blockPoolId) {
Builder builder = this.translator.getBuilder();
if (blockPoolId == null) {
builder.clearBlockPoolId();
} else {
builder.setBlockPoolId(blockPoolId);
}
}
@Override
public void setState(FederationNamenodeServiceState state) {
Builder builder = this.translator.getBuilder();
if (state == null) {
builder.clearState();
} else {
builder.setState(state.toString());
}
}
@Override
public String getRouterId() {
NamenodeMembershipRecordProtoOrBuilder proto =
this.translator.getProtoOrBuilder();
if (!proto.hasRouterId()) {
return null;
}
return proto.getRouterId();
}
@Override
public String getNameserviceId() {
NamenodeMembershipRecordProtoOrBuilder proto =
this.translator.getProtoOrBuilder();
if (!proto.hasNameserviceId()) {
return null;
}
return this.translator.getProtoOrBuilder().getNameserviceId();
}
@Override
public String getNamenodeId() {
NamenodeMembershipRecordProtoOrBuilder proto =
this.translator.getProtoOrBuilder();
if (!proto.hasNamenodeId()) {
return null;
}
return this.translator.getProtoOrBuilder().getNamenodeId();
}
@Override
public String getClusterId() {
NamenodeMembershipRecordProtoOrBuilder proto =
this.translator.getProtoOrBuilder();
if (!proto.hasClusterId()) {
return null;
}
return this.translator.getProtoOrBuilder().getClusterId();
}
@Override
public String getBlockPoolId() {
NamenodeMembershipRecordProtoOrBuilder proto =
this.translator.getProtoOrBuilder();
if (!proto.hasBlockPoolId()) {
return null;
}
return this.translator.getProtoOrBuilder().getBlockPoolId();
}
@Override
public String getRpcAddress() {
NamenodeMembershipRecordProtoOrBuilder proto =
this.translator.getProtoOrBuilder();
if (!proto.hasRpcAddress()) {
return null;
}
return this.translator.getProtoOrBuilder().getRpcAddress();
}
@Override
public String getServiceAddress() {
NamenodeMembershipRecordProtoOrBuilder proto =
this.translator.getProtoOrBuilder();
if (!proto.hasServiceAddress()) {
return null;
}
return this.translator.getProtoOrBuilder().getServiceAddress();
}
@Override
public String getWebAddress() {
NamenodeMembershipRecordProtoOrBuilder proto =
this.translator.getProtoOrBuilder();
if (!proto.hasWebAddress()) {
return null;
}
return this.translator.getProtoOrBuilder().getWebAddress();
}
@Override
public String getLifelineAddress() {
NamenodeMembershipRecordProtoOrBuilder proto =
this.translator.getProtoOrBuilder();
if (!proto.hasLifelineAddress()) {
return null;
}
return this.translator.getProtoOrBuilder().getLifelineAddress();
}
@Override
public boolean getIsSafeMode() {
return this.translator.getProtoOrBuilder().getIsSafeMode();
}
@Override
public FederationNamenodeServiceState getState() {
FederationNamenodeServiceState ret =
FederationNamenodeServiceState.UNAVAILABLE;
NamenodeMembershipRecordProtoOrBuilder proto =
this.translator.getProtoOrBuilder();
if (!proto.hasState()) {
return null;
}
try {
ret = FederationNamenodeServiceState.valueOf(proto.getState());
} catch (IllegalArgumentException e) {
// Ignore this error
}
return ret;
}
@Override
public void setStats(MembershipStats stats) {
if (stats instanceof MembershipStatsPBImpl) {
MembershipStatsPBImpl statsPB = (MembershipStatsPBImpl)stats;
NamenodeMembershipStatsRecordProto statsProto =
(NamenodeMembershipStatsRecordProto)statsPB.getProto();
this.translator.getBuilder().setStats(statsProto);
}
}
@Override
public MembershipStats getStats() throws IOException {
NamenodeMembershipStatsRecordProto statsProto =
this.translator.getProtoOrBuilder().getStats();
MembershipStats stats =
StateStoreSerializer.newRecord(MembershipStats.class);
if (stats instanceof MembershipStatsPBImpl) {
MembershipStatsPBImpl statsPB = (MembershipStatsPBImpl)stats;
statsPB.setProto(statsProto);
return statsPB;
} else {
throw new IOException("Cannot get stats for the membership");
}
}
@Override
public void setLastContact(long contact) {
this.translator.getBuilder().setLastContact(contact);
}
@Override
public long getLastContact() {
return this.translator.getProtoOrBuilder().getLastContact();
}
@Override
public void setDateModified(long time) {
this.translator.getBuilder().setDateModified(time);
}
@Override
public long getDateModified() {
return this.translator.getProtoOrBuilder().getDateModified();
}
@Override
public void setDateCreated(long time) {
this.translator.getBuilder().setDateCreated(time);
}
@Override
public long getDateCreated() {
return this.translator.getProtoOrBuilder().getDateCreated();
}
}

View File

@ -0,0 +1,191 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.records.impl.pb;
import java.io.IOException;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.NamenodeMembershipStatsRecordProto;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.NamenodeMembershipStatsRecordProto.Builder;
import org.apache.hadoop.hdfs.federation.protocol.proto.HdfsServerFederationProtos.NamenodeMembershipStatsRecordProtoOrBuilder;
import org.apache.hadoop.hdfs.server.federation.store.protocol.impl.pb.FederationProtocolPBTranslator;
import org.apache.hadoop.hdfs.server.federation.store.records.MembershipStats;
import com.google.protobuf.Message;
/**
* Protobuf implementation of the MembershipStats record.
*/
public class MembershipStatsPBImpl extends MembershipStats
implements PBRecord {
private FederationProtocolPBTranslator<NamenodeMembershipStatsRecordProto,
Builder, NamenodeMembershipStatsRecordProtoOrBuilder> translator =
new FederationProtocolPBTranslator<NamenodeMembershipStatsRecordProto,
Builder, NamenodeMembershipStatsRecordProtoOrBuilder>(
NamenodeMembershipStatsRecordProto.class);
public MembershipStatsPBImpl() {
}
@Override
public NamenodeMembershipStatsRecordProto getProto() {
return this.translator.build();
}
@Override
public void setProto(Message proto) {
this.translator.setProto(proto);
}
@Override
public void readInstance(String base64String) throws IOException {
this.translator.readInstance(base64String);
}
@Override
public void setTotalSpace(long space) {
this.translator.getBuilder().setTotalSpace(space);
}
@Override
public long getTotalSpace() {
return this.translator.getProtoOrBuilder().getTotalSpace();
}
@Override
public void setAvailableSpace(long space) {
this.translator.getBuilder().setAvailableSpace(space);
}
@Override
public long getAvailableSpace() {
return this.translator.getProtoOrBuilder().getAvailableSpace();
}
@Override
public void setNumOfFiles(long files) {
this.translator.getBuilder().setNumOfFiles(files);
}
@Override
public long getNumOfFiles() {
return this.translator.getProtoOrBuilder().getNumOfFiles();
}
@Override
public void setNumOfBlocks(long blocks) {
this.translator.getBuilder().setNumOfBlocks(blocks);
}
@Override
public long getNumOfBlocks() {
return this.translator.getProtoOrBuilder().getNumOfBlocks();
}
@Override
public void setNumOfBlocksMissing(long blocks) {
this.translator.getBuilder().setNumOfBlocksMissing(blocks);
}
@Override
public long getNumOfBlocksMissing() {
return this.translator.getProtoOrBuilder().getNumOfBlocksMissing();
}
@Override
public void setNumOfBlocksPendingReplication(long blocks) {
this.translator.getBuilder().setNumOfBlocksPendingReplication(blocks);
}
@Override
public long getNumOfBlocksPendingReplication() {
return this.translator.getProtoOrBuilder()
.getNumOfBlocksPendingReplication();
}
@Override
public void setNumOfBlocksUnderReplicated(long blocks) {
this.translator.getBuilder().setNumOfBlocksUnderReplicated(blocks);
}
@Override
public long getNumOfBlocksUnderReplicated() {
return this.translator.getProtoOrBuilder().getNumOfBlocksUnderReplicated();
}
@Override
public void setNumOfBlocksPendingDeletion(long blocks) {
this.translator.getBuilder().setNumOfBlocksPendingDeletion(blocks);
}
@Override
public long getNumOfBlocksPendingDeletion() {
return this.translator.getProtoOrBuilder().getNumOfBlocksPendingDeletion();
}
@Override
public void setNumOfActiveDatanodes(int nodes) {
this.translator.getBuilder().setNumOfActiveDatanodes(nodes);
}
@Override
public int getNumOfActiveDatanodes() {
return this.translator.getProtoOrBuilder().getNumOfActiveDatanodes();
}
@Override
public void setNumOfDeadDatanodes(int nodes) {
this.translator.getBuilder().setNumOfDeadDatanodes(nodes);
}
@Override
public int getNumOfDeadDatanodes() {
return this.translator.getProtoOrBuilder().getNumOfDeadDatanodes();
}
@Override
public void setNumOfDecommissioningDatanodes(int nodes) {
this.translator.getBuilder().setNumOfDecommissioningDatanodes(nodes);
}
@Override
public int getNumOfDecommissioningDatanodes() {
return this.translator.getProtoOrBuilder()
.getNumOfDecommissioningDatanodes();
}
@Override
public void setNumOfDecomActiveDatanodes(int nodes) {
this.translator.getBuilder().setNumOfDecomActiveDatanodes(nodes);
}
@Override
public int getNumOfDecomActiveDatanodes() {
return this.translator.getProtoOrBuilder().getNumOfDecomActiveDatanodes();
}
@Override
public void setNumOfDecomDeadDatanodes(int nodes) {
this.translator.getBuilder().setNumOfDecomDeadDatanodes(nodes);
}
@Override
public int getNumOfDecomDeadDatanodes() {
return this.translator.getProtoOrBuilder().getNumOfDecomDeadDatanodes();
}
}

View File

@ -0,0 +1,107 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
option java_package = "org.apache.hadoop.hdfs.federation.protocol.proto";
option java_outer_classname = "HdfsServerFederationProtos";
option java_generic_services = true;
option java_generate_equals_and_hash = true;
package hadoop.hdfs;
/////////////////////////////////////////////////
// Membership
/////////////////////////////////////////////////
message NamenodeMembershipStatsRecordProto {
optional uint64 totalSpace = 1;
optional uint64 availableSpace = 2;
optional uint64 numOfFiles = 10;
optional uint64 numOfBlocks = 11;
optional uint64 numOfBlocksMissing = 12;
optional uint64 numOfBlocksPendingReplication = 13;
optional uint64 numOfBlocksUnderReplicated = 14;
optional uint64 numOfBlocksPendingDeletion = 15;
optional uint32 numOfActiveDatanodes = 20;
optional uint32 numOfDeadDatanodes = 21;
optional uint32 numOfDecommissioningDatanodes = 22;
optional uint32 numOfDecomActiveDatanodes = 23;
optional uint32 numOfDecomDeadDatanodes = 24;
}
message NamenodeMembershipRecordProto {
optional uint64 dateCreated = 1;
optional uint64 dateModified = 2;
optional uint64 lastContact = 3;
optional string routerId = 4;
optional string nameserviceId = 5;
optional string namenodeId = 6;
optional string clusterId = 7;
optional string blockPoolId = 8;
optional string webAddress = 9;
optional string rpcAddress = 10;
optional string serviceAddress = 11;
optional string lifelineAddress = 12;
optional string state = 13;
optional bool isSafeMode = 14;
optional NamenodeMembershipStatsRecordProto stats = 15;
}
message FederationNamespaceInfoProto {
optional string blockPoolId = 1;
optional string clusterId = 2;
optional string nameserviceId = 3;
}
message GetNamenodeRegistrationsRequestProto {
optional NamenodeMembershipRecordProto membership = 1;
}
message GetNamenodeRegistrationsResponseProto {
repeated NamenodeMembershipRecordProto namenodeMemberships = 1;
}
message GetExpiredRegistrationsRequestProto {
}
message GetNamespaceInfoRequestProto {
}
message GetNamespaceInfoResponseProto {
repeated FederationNamespaceInfoProto namespaceInfos = 1;
}
message UpdateNamenodeRegistrationRequestProto {
optional string nameserviceId = 1;
optional string namenodeId = 2;
optional string state = 3;
}
message UpdateNamenodeRegistrationResponseProto {
optional bool status = 1;
}
message NamenodeHeartbeatRequestProto {
optional NamenodeMembershipRecordProto namenodeMembership = 1;
}
message NamenodeHeartbeatResponseProto {
optional bool status = 1;
}

View File

@ -4755,7 +4755,7 @@
<property>
<name>dfs.federation.router.namenode.resolver.client.class</name>
<value>org.apache.hadoop.hdfs.server.federation.MockResolver</value>
<value>org.apache.hadoop.hdfs.server.federation.resolver.MembershipNamenodeResolver</value>
<description>
Class to resolve the namenode for a subcluster.
</description>
@ -4785,4 +4785,20 @@
</description>
</property>
<property>
<name>dfs.federation.router.cache.ttl</name>
<value>60000</value>
<description>
How often to refresh the State Store caches in milliseconds.
</description>
</property>
<property>
<name>dfs.federation.router.store.membership.expiration</name>
<value>300000</value>
<description>
Expiration time in milliseconds for a membership record.
</description>
</property>
</configuration>

View File

@ -0,0 +1,284 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.resolver;
import static org.apache.hadoop.hdfs.server.federation.FederationTestUtils.NAMENODES;
import static org.apache.hadoop.hdfs.server.federation.FederationTestUtils.NAMESERVICES;
import static org.apache.hadoop.hdfs.server.federation.FederationTestUtils.ROUTERS;
import static org.apache.hadoop.hdfs.server.federation.FederationTestUtils.createNamenodeReport;
import static org.apache.hadoop.hdfs.server.federation.FederationTestUtils.verifyException;
import static org.apache.hadoop.hdfs.server.federation.store.FederationStateStoreTestUtils.clearRecords;
import static org.apache.hadoop.hdfs.server.federation.store.FederationStateStoreTestUtils.getStateStoreConfiguration;
import static org.apache.hadoop.hdfs.server.federation.store.FederationStateStoreTestUtils.newStateStore;
import static org.apache.hadoop.hdfs.server.federation.store.FederationStateStoreTestUtils.waitStateStore;
import static org.junit.Assert.assertEquals;
import static org.junit.Assert.assertFalse;
import static org.junit.Assert.assertNotNull;
import static org.junit.Assert.assertNull;
import static org.junit.Assert.assertTrue;
import java.io.IOException;
import java.util.List;
import java.util.concurrent.TimeUnit;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.ha.HAServiceProtocol.HAServiceState;
import org.apache.hadoop.hdfs.DFSConfigKeys;
import org.apache.hadoop.hdfs.server.federation.store.StateStoreService;
import org.apache.hadoop.hdfs.server.federation.store.StateStoreUnavailableException;
import org.apache.hadoop.hdfs.server.federation.store.records.MembershipState;
import org.junit.AfterClass;
import org.junit.Before;
import org.junit.BeforeClass;
import org.junit.Test;
/**
* Test the basic {@link ActiveNamenodeResolver} functionality.
*/
public class TestNamenodeResolver {
private static StateStoreService stateStore;
private static ActiveNamenodeResolver namenodeResolver;
@BeforeClass
public static void create() throws Exception {
Configuration conf = getStateStoreConfiguration();
// Reduce expirations to 5 seconds
conf.setLong(
DFSConfigKeys.FEDERATION_STORE_MEMBERSHIP_EXPIRATION_MS,
TimeUnit.SECONDS.toMillis(5));
stateStore = newStateStore(conf);
assertNotNull(stateStore);
namenodeResolver = new MembershipNamenodeResolver(conf, stateStore);
namenodeResolver.setRouterId(ROUTERS[0]);
}
@AfterClass
public static void destroy() throws Exception {
stateStore.stop();
stateStore.close();
}
@Before
public void setup() throws IOException, InterruptedException {
// Wait for state store to connect
stateStore.loadDriver();
waitStateStore(stateStore, 10000);
// Clear NN registrations
boolean cleared = clearRecords(stateStore, MembershipState.class);
assertTrue(cleared);
}
@Test
public void testStateStoreDisconnected() throws Exception {
// Add an entry to the store
NamenodeStatusReport report = createNamenodeReport(
NAMESERVICES[0], NAMENODES[0], HAServiceState.ACTIVE);
assertTrue(namenodeResolver.registerNamenode(report));
// Close the data store driver
stateStore.closeDriver();
assertFalse(stateStore.isDriverReady());
// Flush the caches
stateStore.refreshCaches(true);
// Verify commands fail due to no cached data and no state store
// connectivity.
List<? extends FederationNamenodeContext> nns =
namenodeResolver.getNamenodesForBlockPoolId(NAMESERVICES[0]);
assertNull(nns);
verifyException(namenodeResolver, "registerNamenode",
StateStoreUnavailableException.class,
new Class[] {NamenodeStatusReport.class}, new Object[] {report});
}
/**
* Verify the first registration on the resolver.
*
* @param nsId Nameservice identifier.
* @param nnId Namenode identifier within the nemeservice.
* @param resultsCount Number of results expected.
* @param state Expected state for the first one.
* @throws IOException If we cannot get the namenodes.
*/
private void verifyFirstRegistration(String nsId, String nnId,
int resultsCount, FederationNamenodeServiceState state)
throws IOException {
List<? extends FederationNamenodeContext> namenodes =
namenodeResolver.getNamenodesForNameserviceId(nsId);
if (resultsCount == 0) {
assertNull(namenodes);
} else {
assertEquals(resultsCount, namenodes.size());
if (namenodes.size() > 0) {
FederationNamenodeContext namenode = namenodes.get(0);
assertEquals(state, namenode.getState());
assertEquals(nnId, namenode.getNamenodeId());
}
}
}
@Test
public void testRegistrationExpired()
throws InterruptedException, IOException {
// Populate the state store with a single NN element
// 1) ns0:nn0 - Active
// Wait for the entry to expire without heartbeating
// Verify the NN entry is not accessible once expired.
NamenodeStatusReport report = createNamenodeReport(
NAMESERVICES[0], NAMENODES[0], HAServiceState.ACTIVE);
assertTrue(namenodeResolver.registerNamenode(report));
// Load cache
stateStore.refreshCaches(true);
// Verify
verifyFirstRegistration(
NAMESERVICES[0], NAMENODES[0], 1,
FederationNamenodeServiceState.ACTIVE);
// Wait past expiration (set in conf to 5 seconds)
Thread.sleep(6000);
// Reload cache
stateStore.refreshCaches(true);
// Verify entry is now expired and is no longer in the cache
verifyFirstRegistration(
NAMESERVICES[0], NAMENODES[0], 0,
FederationNamenodeServiceState.ACTIVE);
// Heartbeat again, updates dateModified
assertTrue(namenodeResolver.registerNamenode(report));
// Reload cache
stateStore.refreshCaches(true);
// Verify updated entry is marked active again and accessible to RPC server
verifyFirstRegistration(
NAMESERVICES[0], NAMENODES[0], 1,
FederationNamenodeServiceState.ACTIVE);
}
@Test
public void testRegistrationNamenodeSelection()
throws InterruptedException, IOException {
// 1) ns0:nn0 - Active
// 2) ns0:nn1 - Standby (newest)
// Verify the selected entry is the active entry
assertTrue(namenodeResolver.registerNamenode(
createNamenodeReport(
NAMESERVICES[0], NAMENODES[0], HAServiceState.ACTIVE)));
Thread.sleep(100);
assertTrue(namenodeResolver.registerNamenode(
createNamenodeReport(
NAMESERVICES[0], NAMENODES[1], HAServiceState.STANDBY)));
stateStore.refreshCaches(true);
verifyFirstRegistration(
NAMESERVICES[0], NAMENODES[0], 2,
FederationNamenodeServiceState.ACTIVE);
// 1) ns0:nn0 - Expired (stale)
// 2) ns0:nn1 - Standby (newest)
// Verify the selected entry is the standby entry as the active entry is
// stale
assertTrue(namenodeResolver.registerNamenode(
createNamenodeReport(
NAMESERVICES[0], NAMENODES[0], HAServiceState.ACTIVE)));
// Expire active registration
Thread.sleep(6000);
// Refresh standby registration
assertTrue(namenodeResolver.registerNamenode(createNamenodeReport(
NAMESERVICES[0], NAMENODES[1], HAServiceState.STANDBY)));
// Verify that standby is selected (active is now expired)
stateStore.refreshCaches(true);
verifyFirstRegistration(NAMESERVICES[0], NAMENODES[1], 1,
FederationNamenodeServiceState.STANDBY);
// 1) ns0:nn0 - Active
// 2) ns0:nn1 - Unavailable (newest)
// Verify the selected entry is the active entry
assertTrue(namenodeResolver.registerNamenode(createNamenodeReport(
NAMESERVICES[0], NAMENODES[0], HAServiceState.ACTIVE)));
Thread.sleep(100);
assertTrue(namenodeResolver.registerNamenode(createNamenodeReport(
NAMESERVICES[0], NAMENODES[1], null)));
stateStore.refreshCaches(true);
verifyFirstRegistration(NAMESERVICES[0], NAMENODES[0], 2,
FederationNamenodeServiceState.ACTIVE);
// 1) ns0:nn0 - Unavailable (newest)
// 2) ns0:nn1 - Standby
// Verify the selected entry is the standby entry
assertTrue(namenodeResolver.registerNamenode(createNamenodeReport(
NAMESERVICES[0], NAMENODES[1], HAServiceState.STANDBY)));
Thread.sleep(1000);
assertTrue(namenodeResolver.registerNamenode(createNamenodeReport(
NAMESERVICES[0], NAMENODES[0], null)));
stateStore.refreshCaches(true);
verifyFirstRegistration(NAMESERVICES[0], NAMENODES[1], 2,
FederationNamenodeServiceState.STANDBY);
// 1) ns0:nn0 - Active (oldest)
// 2) ns0:nn1 - Standby
// 3) ns0:nn2 - Active (newest)
// Verify the selected entry is the newest active entry
assertTrue(namenodeResolver.registerNamenode(
createNamenodeReport(NAMESERVICES[0], NAMENODES[0], null)));
Thread.sleep(100);
assertTrue(namenodeResolver.registerNamenode(createNamenodeReport(
NAMESERVICES[0], NAMENODES[1], HAServiceState.STANDBY)));
Thread.sleep(100);
assertTrue(namenodeResolver.registerNamenode(createNamenodeReport(
NAMESERVICES[0], NAMENODES[2], HAServiceState.ACTIVE)));
stateStore.refreshCaches(true);
verifyFirstRegistration(NAMESERVICES[0], NAMENODES[2], 3,
FederationNamenodeServiceState.ACTIVE);
// 1) ns0:nn0 - Standby (oldest)
// 2) ns0:nn1 - Standby (newest)
// 3) ns0:nn2 - Standby
// Verify the selected entry is the newest standby entry
assertTrue(namenodeResolver.registerNamenode(createNamenodeReport(
NAMESERVICES[0], NAMENODES[0], HAServiceState.STANDBY)));
assertTrue(namenodeResolver.registerNamenode(createNamenodeReport(
NAMESERVICES[0], NAMENODES[2], HAServiceState.STANDBY)));
Thread.sleep(1500);
assertTrue(namenodeResolver.registerNamenode(createNamenodeReport(
NAMESERVICES[0], NAMENODES[1], HAServiceState.STANDBY)));
stateStore.refreshCaches(true);
verifyFirstRegistration(NAMESERVICES[0], NAMENODES[1], 3,
FederationNamenodeServiceState.STANDBY);
}
}

View File

@ -34,9 +34,12 @@ import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.fs.CommonConfigurationKeysPublic;
import org.apache.hadoop.hdfs.DFSConfigKeys;
import org.apache.hadoop.hdfs.HdfsConfiguration;
import org.apache.hadoop.hdfs.server.federation.resolver.FederationNamenodeServiceState;
import org.apache.hadoop.hdfs.server.federation.store.driver.StateStoreDriver;
import org.apache.hadoop.hdfs.server.federation.store.driver.impl.StateStoreFileBaseImpl;
import org.apache.hadoop.hdfs.server.federation.store.records.BaseRecord;
import org.apache.hadoop.hdfs.server.federation.store.records.MembershipState;
import org.apache.hadoop.hdfs.server.federation.store.records.MembershipStats;
import org.apache.hadoop.util.Time;
/**
@ -96,7 +99,7 @@ public final class FederationStateStoreTestUtils {
* @throws IOException If it cannot create the State Store.
* @throws InterruptedException If we cannot wait for the store to start.
*/
public static StateStoreService getStateStore(
public static StateStoreService newStateStore(
Configuration configuration) throws IOException, InterruptedException {
StateStoreService stateStore = new StateStoreService();
@ -205,6 +208,7 @@ public final class FederationStateStoreTestUtils {
if (!synchronizeRecords(store, emptyList, recordClass)) {
return false;
}
store.refreshCaches(true);
return true;
}
@ -229,4 +233,21 @@ public final class FederationStateStoreTestUtils {
}
return false;
}
public static MembershipState createMockRegistrationForNamenode(
String nameserviceId, String namenodeId,
FederationNamenodeServiceState state) throws IOException {
MembershipState entry = MembershipState.newInstance(
"routerId", nameserviceId, namenodeId, "clusterId", "test",
"0.0.0.0:0", "0.0.0.0:0", "0.0.0.0:0", "0.0.0.0:0", state, false);
MembershipStats stats = MembershipStats.newInstance();
stats.setNumOfActiveDatanodes(100);
stats.setNumOfDeadDatanodes(10);
stats.setNumOfDecommissioningDatanodes(20);
stats.setNumOfDecomActiveDatanodes(15);
stats.setNumOfDecomDeadDatanodes(5);
stats.setNumOfBlocks(10);
entry.setStats(stats);
return entry;
}
}

View File

@ -0,0 +1,81 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store;
import static org.apache.hadoop.hdfs.server.federation.store.FederationStateStoreTestUtils.newStateStore;
import static org.apache.hadoop.hdfs.server.federation.store.FederationStateStoreTestUtils.getStateStoreConfiguration;
import static org.apache.hadoop.hdfs.server.federation.store.FederationStateStoreTestUtils.waitStateStore;
import static org.junit.Assert.assertNotNull;
import java.io.IOException;
import java.util.concurrent.TimeUnit;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hdfs.DFSConfigKeys;
import org.junit.AfterClass;
import org.junit.Before;
import org.junit.BeforeClass;
/**
* Test the basic {@link StateStoreService} {@link MountTableStore}
* functionality.
*/
public class TestStateStoreBase {
private static StateStoreService stateStore;
private static Configuration conf;
protected static StateStoreService getStateStore() {
return stateStore;
}
protected static Configuration getConf() {
return conf;
}
@BeforeClass
public static void createBase() throws IOException, InterruptedException {
conf = getStateStoreConfiguration();
// Disable auto-reconnect to data store
conf.setLong(DFSConfigKeys.FEDERATION_STORE_CONNECTION_TEST_MS,
TimeUnit.HOURS.toMillis(1));
}
@AfterClass
public static void destroyBase() throws Exception {
if (stateStore != null) {
stateStore.stop();
stateStore.close();
stateStore = null;
}
}
@Before
public void setupBase() throws IOException, InterruptedException,
InstantiationException, IllegalAccessException {
if (stateStore == null) {
stateStore = newStateStore(conf);
assertNotNull(stateStore);
}
// Wait for state store to connect
stateStore.loadDriver();
waitStateStore(stateStore, TimeUnit.SECONDS.toMillis(10));
}
}

View File

@ -0,0 +1,463 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store;
import static org.apache.hadoop.hdfs.server.federation.FederationTestUtils.NAMENODES;
import static org.apache.hadoop.hdfs.server.federation.FederationTestUtils.NAMESERVICES;
import static org.apache.hadoop.hdfs.server.federation.FederationTestUtils.ROUTERS;
import static org.apache.hadoop.hdfs.server.federation.FederationTestUtils.verifyException;
import static org.apache.hadoop.hdfs.server.federation.store.FederationStateStoreTestUtils.clearRecords;
import static org.apache.hadoop.hdfs.server.federation.store.FederationStateStoreTestUtils.createMockRegistrationForNamenode;
import static org.apache.hadoop.hdfs.server.federation.store.FederationStateStoreTestUtils.synchronizeRecords;
import static org.junit.Assert.assertEquals;
import static org.junit.Assert.assertFalse;
import static org.junit.Assert.assertNotNull;
import static org.junit.Assert.assertNull;
import static org.junit.Assert.assertTrue;
import java.io.IOException;
import java.util.ArrayList;
import java.util.List;
import java.util.concurrent.TimeUnit;
import org.apache.hadoop.hdfs.DFSConfigKeys;
import org.apache.hadoop.hdfs.server.federation.resolver.FederationNamenodeServiceState;
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetNamenodeRegistrationsRequest;
import org.apache.hadoop.hdfs.server.federation.store.protocol.GetNamenodeRegistrationsResponse;
import org.apache.hadoop.hdfs.server.federation.store.protocol.NamenodeHeartbeatRequest;
import org.apache.hadoop.hdfs.server.federation.store.protocol.NamenodeHeartbeatResponse;
import org.apache.hadoop.hdfs.server.federation.store.protocol.UpdateNamenodeRegistrationRequest;
import org.apache.hadoop.hdfs.server.federation.store.records.MembershipState;
import org.apache.hadoop.util.Time;
import org.junit.Before;
import org.junit.BeforeClass;
import org.junit.Test;
/**
* Test the basic {@link MembershipStore} membership functionality.
*/
public class TestStateStoreMembershipState extends TestStateStoreBase {
private static MembershipStore membershipStore;
@BeforeClass
public static void create() {
// Reduce expirations to 5 seconds
getConf().setLong(
DFSConfigKeys.FEDERATION_STORE_MEMBERSHIP_EXPIRATION_MS,
TimeUnit.SECONDS.toMillis(5));
}
@Before
public void setup() throws IOException, InterruptedException {
membershipStore =
getStateStore().getRegisteredRecordStore(MembershipStore.class);
// Clear NN registrations
assertTrue(clearRecords(getStateStore(), MembershipState.class));
}
@Test
public void testNamenodeStateOverride() throws Exception {
// Populate the state store
// 1) ns0:nn0 - Standby
String ns = "ns0";
String nn = "nn0";
MembershipState report = createRegistration(
ns, nn, ROUTERS[1], FederationNamenodeServiceState.STANDBY);
assertTrue(namenodeHeartbeat(report));
// Load data into cache and calculate quorum
assertTrue(getStateStore().loadCache(MembershipStore.class, true));
MembershipState existingState = getNamenodeRegistration(ns, nn);
assertEquals(
FederationNamenodeServiceState.STANDBY, existingState.getState());
// Override cache
UpdateNamenodeRegistrationRequest request =
UpdateNamenodeRegistrationRequest.newInstance(
ns, nn, FederationNamenodeServiceState.ACTIVE);
assertTrue(membershipStore.updateNamenodeRegistration(request).getResult());
MembershipState newState = getNamenodeRegistration(ns, nn);
assertEquals(FederationNamenodeServiceState.ACTIVE, newState.getState());
}
@Test
public void testStateStoreDisconnected() throws Exception {
// Close the data store driver
getStateStore().closeDriver();
assertFalse(getStateStore().isDriverReady());
NamenodeHeartbeatRequest hbRequest = NamenodeHeartbeatRequest.newInstance();
hbRequest.setNamenodeMembership(
createMockRegistrationForNamenode(
"test", "test", FederationNamenodeServiceState.UNAVAILABLE));
verifyException(membershipStore, "namenodeHeartbeat",
StateStoreUnavailableException.class,
new Class[] {NamenodeHeartbeatRequest.class},
new Object[] {hbRequest });
// Information from cache, no exception should be triggered for these
// TODO - should cached info expire at some point?
GetNamenodeRegistrationsRequest getRequest =
GetNamenodeRegistrationsRequest.newInstance();
verifyException(membershipStore,
"getNamenodeRegistrations", null,
new Class[] {GetNamenodeRegistrationsRequest.class},
new Object[] {getRequest});
verifyException(membershipStore,
"getExpiredNamenodeRegistrations", null,
new Class[] {GetNamenodeRegistrationsRequest.class},
new Object[] {getRequest});
UpdateNamenodeRegistrationRequest overrideRequest =
UpdateNamenodeRegistrationRequest.newInstance();
verifyException(membershipStore,
"updateNamenodeRegistration", null,
new Class[] {UpdateNamenodeRegistrationRequest.class},
new Object[] {overrideRequest});
}
private void registerAndLoadRegistrations(
List<MembershipState> registrationList) throws IOException {
// Populate
assertTrue(synchronizeRecords(
getStateStore(), registrationList, MembershipState.class));
// Load into cache
assertTrue(getStateStore().loadCache(MembershipStore.class, true));
}
private MembershipState createRegistration(String ns, String nn,
String router, FederationNamenodeServiceState state) throws IOException {
MembershipState record = MembershipState.newInstance(
router, ns,
nn, "testcluster", "testblock-" + ns, "testrpc-"+ ns + nn,
"testservice-"+ ns + nn, "testlifeline-"+ ns + nn,
"testweb-" + ns + nn, state, false);
return record;
}
@Test
public void testRegistrationMajorityQuorum()
throws InterruptedException, IOException {
// Populate the state store with a set of non-matching elements
// 1) ns0:nn0 - Standby (newest)
// 2) ns0:nn0 - Active (oldest)
// 3) ns0:nn0 - Active (2nd oldest)
// 4) ns0:nn0 - Active (3nd oldest element, newest active element)
// Verify the selected entry is the newest majority opinion (4)
String ns = "ns0";
String nn = "nn0";
// Active - oldest
MembershipState report = createRegistration(
ns, nn, ROUTERS[1], FederationNamenodeServiceState.ACTIVE);
assertTrue(namenodeHeartbeat(report));
Thread.sleep(1000);
// Active - 2nd oldest
report = createRegistration(
ns, nn, ROUTERS[2], FederationNamenodeServiceState.ACTIVE);
assertTrue(namenodeHeartbeat(report));
Thread.sleep(1000);
// Active - 3rd oldest, newest active element
report = createRegistration(
ns, nn, ROUTERS[3], FederationNamenodeServiceState.ACTIVE);
assertTrue(namenodeHeartbeat(report));
// standby - newest overall
report = createRegistration(
ns, nn, ROUTERS[0], FederationNamenodeServiceState.STANDBY);
assertTrue(namenodeHeartbeat(report));
// Load and calculate quorum
assertTrue(getStateStore().loadCache(MembershipStore.class, true));
// Verify quorum entry
MembershipState quorumEntry = getNamenodeRegistration(
report.getNameserviceId(), report.getNamenodeId());
assertNotNull(quorumEntry);
assertEquals(quorumEntry.getRouterId(), ROUTERS[3]);
}
@Test
public void testRegistrationQuorumExcludesExpired()
throws InterruptedException, IOException {
// Populate the state store with some expired entries and verify the expired
// entries are ignored.
// 1) ns0:nn0 - Active
// 2) ns0:nn0 - Expired
// 3) ns0:nn0 - Expired
// 4) ns0:nn0 - Expired
// Verify the selected entry is the active entry
List<MembershipState> registrationList = new ArrayList<>();
String ns = "ns0";
String nn = "nn0";
String rpcAddress = "testrpcaddress";
String serviceAddress = "testserviceaddress";
String lifelineAddress = "testlifelineaddress";
String blockPoolId = "testblockpool";
String clusterId = "testcluster";
String webAddress = "testwebaddress";
boolean safemode = false;
// Active
MembershipState record = MembershipState.newInstance(
ROUTERS[0], ns, nn, clusterId, blockPoolId,
rpcAddress, serviceAddress, lifelineAddress, webAddress,
FederationNamenodeServiceState.ACTIVE, safemode);
registrationList.add(record);
// Expired
record = MembershipState.newInstance(
ROUTERS[1], ns, nn, clusterId, blockPoolId,
rpcAddress, serviceAddress, lifelineAddress, webAddress,
FederationNamenodeServiceState.EXPIRED, safemode);
registrationList.add(record);
// Expired
record = MembershipState.newInstance(
ROUTERS[2], ns, nn, clusterId, blockPoolId,
rpcAddress, serviceAddress, lifelineAddress, webAddress,
FederationNamenodeServiceState.EXPIRED, safemode);
registrationList.add(record);
// Expired
record = MembershipState.newInstance(
ROUTERS[3], ns, nn, clusterId, blockPoolId,
rpcAddress, serviceAddress, lifelineAddress, webAddress,
FederationNamenodeServiceState.EXPIRED, safemode);
registrationList.add(record);
registerAndLoadRegistrations(registrationList);
// Verify quorum entry chooses active membership
MembershipState quorumEntry = getNamenodeRegistration(
record.getNameserviceId(), record.getNamenodeId());
assertNotNull(quorumEntry);
assertEquals(ROUTERS[0], quorumEntry.getRouterId());
}
@Test
public void testRegistrationQuorumAllExpired() throws IOException {
// 1) ns0:nn0 - Expired (oldest)
// 2) ns0:nn0 - Expired
// 3) ns0:nn0 - Expired
// 4) ns0:nn0 - Expired
// Verify no entry is either selected or cached
List<MembershipState> registrationList = new ArrayList<>();
String ns = NAMESERVICES[0];
String nn = NAMENODES[0];
String rpcAddress = "testrpcaddress";
String serviceAddress = "testserviceaddress";
String lifelineAddress = "testlifelineaddress";
String blockPoolId = "testblockpool";
String clusterId = "testcluster";
String webAddress = "testwebaddress";
boolean safemode = false;
long startingTime = Time.now();
// Expired
MembershipState record = MembershipState.newInstance(
ROUTERS[0], ns, nn, clusterId, blockPoolId,
rpcAddress, webAddress, lifelineAddress, webAddress,
FederationNamenodeServiceState.EXPIRED, safemode);
record.setDateModified(startingTime - 10000);
registrationList.add(record);
// Expired
record = MembershipState.newInstance(
ROUTERS[1], ns, nn, clusterId, blockPoolId,
rpcAddress, serviceAddress, lifelineAddress, webAddress,
FederationNamenodeServiceState.EXPIRED, safemode);
record.setDateModified(startingTime);
registrationList.add(record);
// Expired
record = MembershipState.newInstance(
ROUTERS[2], ns, nn, clusterId, blockPoolId,
rpcAddress, serviceAddress, lifelineAddress, webAddress,
FederationNamenodeServiceState.EXPIRED, safemode);
record.setDateModified(startingTime);
registrationList.add(record);
// Expired
record = MembershipState.newInstance(
ROUTERS[3], ns, nn, clusterId, blockPoolId,
rpcAddress, serviceAddress, lifelineAddress, webAddress,
FederationNamenodeServiceState.EXPIRED, safemode);
record.setDateModified(startingTime);
registrationList.add(record);
registerAndLoadRegistrations(registrationList);
// Verify no entry is found for this nameservice
assertNull(getNamenodeRegistration(
record.getNameserviceId(), record.getNamenodeId()));
}
@Test
public void testRegistrationNoQuorum()
throws InterruptedException, IOException {
// Populate the state store with a set of non-matching elements
// 1) ns0:nn0 - Standby (newest)
// 2) ns0:nn0 - Standby (oldest)
// 3) ns0:nn0 - Active (2nd oldest)
// 4) ns0:nn0 - Active (3nd oldest element, newest active element)
// Verify the selected entry is the newest entry (1)
MembershipState report1 = createRegistration(
NAMESERVICES[0], NAMENODES[0], ROUTERS[1],
FederationNamenodeServiceState.STANDBY);
assertTrue(namenodeHeartbeat(report1));
Thread.sleep(100);
MembershipState report2 = createRegistration(
NAMESERVICES[0], NAMENODES[0], ROUTERS[2],
FederationNamenodeServiceState.ACTIVE);
assertTrue(namenodeHeartbeat(report2));
Thread.sleep(100);
MembershipState report3 = createRegistration(
NAMESERVICES[0], NAMENODES[0], ROUTERS[3],
FederationNamenodeServiceState.ACTIVE);
assertTrue(namenodeHeartbeat(report3));
Thread.sleep(100);
MembershipState report4 = createRegistration(
NAMESERVICES[0], NAMENODES[0], ROUTERS[0],
FederationNamenodeServiceState.STANDBY);
assertTrue(namenodeHeartbeat(report4));
// Load and calculate quorum
assertTrue(getStateStore().loadCache(MembershipStore.class, true));
// Verify quorum entry uses the newest data, even though it is standby
MembershipState quorumEntry = getNamenodeRegistration(
report1.getNameserviceId(), report1.getNamenodeId());
assertNotNull(quorumEntry);
assertEquals(ROUTERS[0], quorumEntry.getRouterId());
assertEquals(
FederationNamenodeServiceState.STANDBY, quorumEntry.getState());
}
@Test
public void testRegistrationExpired()
throws InterruptedException, IOException {
// Populate the state store with a single NN element
// 1) ns0:nn0 - Active
// Wait for the entry to expire without heartbeating
// Verify the NN entry is populated as EXPIRED internally in the state store
MembershipState report = createRegistration(
NAMESERVICES[0], NAMENODES[0], ROUTERS[0],
FederationNamenodeServiceState.ACTIVE);
assertTrue(namenodeHeartbeat(report));
// Load cache
assertTrue(getStateStore().loadCache(MembershipStore.class, true));
// Verify quorum and entry
MembershipState quorumEntry = getNamenodeRegistration(
report.getNameserviceId(), report.getNamenodeId());
assertNotNull(quorumEntry);
assertEquals(ROUTERS[0], quorumEntry.getRouterId());
assertEquals(FederationNamenodeServiceState.ACTIVE, quorumEntry.getState());
// Wait past expiration (set in conf to 5 seconds)
Thread.sleep(6000);
// Reload cache
assertTrue(getStateStore().loadCache(MembershipStore.class, true));
// Verify entry is now expired and is no longer in the cache
quorumEntry = getNamenodeRegistration(NAMESERVICES[0], NAMENODES[0]);
assertNull(quorumEntry);
// Verify entry is now expired and can't be used by RPC service
quorumEntry = getNamenodeRegistration(
report.getNameserviceId(), report.getNamenodeId());
assertNull(quorumEntry);
// Heartbeat again, updates dateModified
assertTrue(namenodeHeartbeat(report));
// Reload cache
assertTrue(getStateStore().loadCache(MembershipStore.class, true));
// Verify updated entry marked as active and is accessible to RPC server
quorumEntry = getNamenodeRegistration(
report.getNameserviceId(), report.getNamenodeId());
assertNotNull(quorumEntry);
assertEquals(ROUTERS[0], quorumEntry.getRouterId());
assertEquals(FederationNamenodeServiceState.ACTIVE, quorumEntry.getState());
}
/**
* Get a single namenode membership record from the store.
*
* @param nsId The HDFS nameservice ID to search for
* @param nnId The HDFS namenode ID to search for
* @return The single NamenodeMembershipRecord that matches the query or null
* if not found.
* @throws IOException if the query could not be executed.
*/
private MembershipState getNamenodeRegistration(
final String nsId, final String nnId) throws IOException {
MembershipState partial = MembershipState.newInstance();
partial.setNameserviceId(nsId);
partial.setNamenodeId(nnId);
GetNamenodeRegistrationsRequest request =
GetNamenodeRegistrationsRequest.newInstance(partial);
GetNamenodeRegistrationsResponse response =
membershipStore.getNamenodeRegistrations(request);
List<MembershipState> results = response.getNamenodeMemberships();
if (results != null && results.size() == 1) {
MembershipState record = results.get(0);
return record;
}
return null;
}
/**
* Register a namenode heartbeat with the state store.
*
* @param store FederationMembershipStateStore instance to retrieve the
* membership data records.
* @param namenode A fully populated namenode membership record to be
* committed to the data store.
* @return True if successful, false otherwise.
* @throws IOException if the state store query could not be performed.
*/
private boolean namenodeHeartbeat(MembershipState namenode)
throws IOException {
NamenodeHeartbeatRequest request =
NamenodeHeartbeatRequest.newInstance(namenode);
NamenodeHeartbeatResponse response =
membershipStore.namenodeHeartbeat(request);
return response.getResult();
}
}

View File

@ -31,11 +31,14 @@ import java.util.HashMap;
import java.util.List;
import java.util.Map;
import java.util.Map.Entry;
import java.util.Random;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hdfs.server.federation.resolver.FederationNamenodeServiceState;
import org.apache.hadoop.hdfs.server.federation.store.FederationStateStoreTestUtils;
import org.apache.hadoop.hdfs.server.federation.store.StateStoreService;
import org.apache.hadoop.hdfs.server.federation.store.records.BaseRecord;
import org.apache.hadoop.hdfs.server.federation.store.records.MembershipState;
import org.apache.hadoop.hdfs.server.federation.store.records.Query;
import org.apache.hadoop.hdfs.server.federation.store.records.QueryResult;
import org.junit.AfterClass;
@ -54,6 +57,8 @@ public class TestStateStoreDriverBase {
private static StateStoreService stateStore;
private static Configuration conf;
private static final Random RANDOM = new Random();
/**
* Get the State Store driver.
@ -78,29 +83,47 @@ public class TestStateStoreDriverBase {
*/
public static void getStateStore(Configuration config) throws Exception {
conf = config;
stateStore = FederationStateStoreTestUtils.getStateStore(conf);
stateStore = FederationStateStoreTestUtils.newStateStore(conf);
}
private String generateRandomString() {
String randomString = "/randomString-" + RANDOM.nextInt();
return randomString;
}
@SuppressWarnings("rawtypes")
private <T extends Enum> T generateRandomEnum(Class<T> enumClass) {
int x = RANDOM.nextInt(enumClass.getEnumConstants().length);
T data = enumClass.getEnumConstants()[x];
return data;
}
@SuppressWarnings("unchecked")
private <T extends BaseRecord> T generateFakeRecord(Class<T> recordClass)
throws IllegalArgumentException, IllegalAccessException, IOException {
// TODO add record
if (recordClass == MembershipState.class) {
return (T) MembershipState.newInstance(generateRandomString(),
generateRandomString(), generateRandomString(),
generateRandomString(), generateRandomString(),
generateRandomString(), generateRandomString(),
generateRandomString(), generateRandomString(),
generateRandomEnum(FederationNamenodeServiceState.class), false);
}
return null;
}
/**
* Validate if a record is the same.
*
* @param original
* @param committed
* @param original Original record.
* @param committed Committed record.
* @param assertEquals Assert if the records are equal or just return.
* @return
* @throws IllegalArgumentException
* @throws IllegalAccessException
* @return If the record is successfully validated.
*/
private boolean validateRecord(
BaseRecord original, BaseRecord committed, boolean assertEquals)
throws IllegalArgumentException, IllegalAccessException {
BaseRecord original, BaseRecord committed, boolean assertEquals) {
boolean ret = true;
@ -131,7 +154,7 @@ public class TestStateStoreDriverBase {
}
public static void removeAll(StateStoreDriver driver) throws IOException {
// TODO add records to remove
driver.removeAll(MembershipState.class);
}
public <T extends BaseRecord> void testInsert(
@ -139,17 +162,20 @@ public class TestStateStoreDriverBase {
throws IllegalArgumentException, IllegalAccessException, IOException {
assertTrue(driver.removeAll(recordClass));
QueryResult<T> records = driver.get(recordClass);
assertTrue(records.getRecords().isEmpty());
QueryResult<T> queryResult0 = driver.get(recordClass);
List<T> records0 = queryResult0.getRecords();
assertTrue(records0.isEmpty());
// Insert single
BaseRecord record = generateFakeRecord(recordClass);
driver.put(record, true, false);
// Verify
records = driver.get(recordClass);
assertEquals(1, records.getRecords().size());
validateRecord(record, records.getRecords().get(0), true);
QueryResult<T> queryResult1 = driver.get(recordClass);
List<T> records1 = queryResult1.getRecords();
assertEquals(1, records1.size());
T record0 = records1.get(0);
validateRecord(record, record0, true);
// Insert multiple
List<T> insertList = new ArrayList<>();
@ -160,8 +186,9 @@ public class TestStateStoreDriverBase {
driver.putAll(insertList, true, false);
// Verify
records = driver.get(recordClass);
assertEquals(11, records.getRecords().size());
QueryResult<T> queryResult2 = driver.get(recordClass);
List<T> records2 = queryResult2.getRecords();
assertEquals(11, records2.size());
}
public <T extends BaseRecord> void testFetchErrors(StateStoreDriver driver,
@ -319,23 +346,23 @@ public class TestStateStoreDriverBase {
public void testInsert(StateStoreDriver driver)
throws IllegalArgumentException, IllegalAccessException, IOException {
// TODO add records
testInsert(driver, MembershipState.class);
}
public void testPut(StateStoreDriver driver)
throws IllegalArgumentException, ReflectiveOperationException,
IOException, SecurityException {
// TODO add records
testPut(driver, MembershipState.class);
}
public void testRemove(StateStoreDriver driver)
throws IllegalArgumentException, IllegalAccessException, IOException {
// TODO add records
testRemove(driver, MembershipState.class);
}
public void testFetchErrors(StateStoreDriver driver)
throws IllegalArgumentException, IllegalAccessException, IOException {
// TODO add records
testFetchErrors(driver, MembershipState.class);
}
/**

View File

@ -0,0 +1,129 @@
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hdfs.server.federation.store.records;
import static org.junit.Assert.assertEquals;
import java.io.IOException;
import org.apache.hadoop.hdfs.server.federation.resolver.FederationNamenodeServiceState;
import org.apache.hadoop.hdfs.server.federation.store.driver.StateStoreSerializer;
import org.junit.Test;
/**
* Test the Membership State records.
*/
public class TestMembershipState {
private static final String ROUTER = "router";
private static final String NAMESERVICE = "nameservice";
private static final String NAMENODE = "namenode";
private static final String CLUSTER_ID = "cluster";
private static final String BLOCKPOOL_ID = "blockpool";
private static final String RPC_ADDRESS = "rpcaddress";
private static final String SERVICE_ADDRESS = "serviceaddress";
private static final String LIFELINE_ADDRESS = "lifelineaddress";
private static final String WEB_ADDRESS = "webaddress";
private static final boolean SAFE_MODE = false;
private static final long DATE_CREATED = 100;
private static final long DATE_MODIFIED = 200;
private static final long NUM_BLOCKS = 300;
private static final long NUM_FILES = 400;
private static final int NUM_DEAD = 500;
private static final int NUM_ACTIVE = 600;
private static final int NUM_DECOM = 700;
private static final int NUM_DECOM_ACTIVE = 800;
private static final int NUM_DECOM_DEAD = 900;
private static final long NUM_BLOCK_MISSING = 1000;
private static final long TOTAL_SPACE = 1100;
private static final long AVAILABLE_SPACE = 1200;
private static final FederationNamenodeServiceState STATE =
FederationNamenodeServiceState.ACTIVE;
private MembershipState createRecord() throws IOException {
MembershipState record = MembershipState.newInstance(
ROUTER, NAMESERVICE, NAMENODE, CLUSTER_ID,
BLOCKPOOL_ID, RPC_ADDRESS, SERVICE_ADDRESS, LIFELINE_ADDRESS,
WEB_ADDRESS, STATE, SAFE_MODE);
record.setDateCreated(DATE_CREATED);
record.setDateModified(DATE_MODIFIED);
MembershipStats stats = MembershipStats.newInstance();
stats.setNumOfBlocks(NUM_BLOCKS);
stats.setNumOfFiles(NUM_FILES);
stats.setNumOfActiveDatanodes(NUM_ACTIVE);
stats.setNumOfDeadDatanodes(NUM_DEAD);
stats.setNumOfDecommissioningDatanodes(NUM_DECOM);
stats.setNumOfDecomActiveDatanodes(NUM_DECOM_ACTIVE);
stats.setNumOfDecomDeadDatanodes(NUM_DECOM_DEAD);
stats.setNumOfBlocksMissing(NUM_BLOCK_MISSING);
stats.setTotalSpace(TOTAL_SPACE);
stats.setAvailableSpace(AVAILABLE_SPACE);
record.setStats(stats);
return record;
}
private void validateRecord(MembershipState record) throws IOException {
assertEquals(ROUTER, record.getRouterId());
assertEquals(NAMESERVICE, record.getNameserviceId());
assertEquals(CLUSTER_ID, record.getClusterId());
assertEquals(BLOCKPOOL_ID, record.getBlockPoolId());
assertEquals(RPC_ADDRESS, record.getRpcAddress());
assertEquals(WEB_ADDRESS, record.getWebAddress());
assertEquals(STATE, record.getState());
assertEquals(SAFE_MODE, record.getIsSafeMode());
assertEquals(DATE_CREATED, record.getDateCreated());
assertEquals(DATE_MODIFIED, record.getDateModified());
MembershipStats stats = record.getStats();
assertEquals(NUM_BLOCKS, stats.getNumOfBlocks());
assertEquals(NUM_FILES, stats.getNumOfFiles());
assertEquals(NUM_ACTIVE, stats.getNumOfActiveDatanodes());
assertEquals(NUM_DEAD, stats.getNumOfDeadDatanodes());
assertEquals(NUM_DECOM, stats.getNumOfDecommissioningDatanodes());
assertEquals(NUM_DECOM_ACTIVE, stats.getNumOfDecomActiveDatanodes());
assertEquals(NUM_DECOM_DEAD, stats.getNumOfDecomDeadDatanodes());
assertEquals(TOTAL_SPACE, stats.getTotalSpace());
assertEquals(AVAILABLE_SPACE, stats.getAvailableSpace());
}
@Test
public void testGetterSetter() throws IOException {
MembershipState record = createRecord();
validateRecord(record);
}
@Test
public void testSerialization() throws IOException {
MembershipState record = createRecord();
StateStoreSerializer serializer = StateStoreSerializer.getSerializer();
String serializedString = serializer.serializeString(record);
MembershipState newRecord =
serializer.deserialize(serializedString, MembershipState.class);
validateRecord(newRecord);
}
}