|
@@ -17,65 +17,71 @@
|
|
|
*/
|
|
|
package org.apache.hadoop.ozone;
|
|
|
|
|
|
+import com.google.common.annotations.VisibleForTesting;
|
|
|
import com.google.common.base.Preconditions;
|
|
|
-import org.apache.hadoop.classification.InterfaceAudience;
|
|
|
+import org.apache.hadoop.conf.Configurable;
|
|
|
import org.apache.hadoop.conf.Configuration;
|
|
|
import org.apache.hadoop.hdds.scm.ScmConfigKeys;
|
|
|
import org.apache.hadoop.hdfs.server.datanode.DataNode;
|
|
|
-import org.apache.hadoop.hdfs.server.datanode.DataNodeServicePlugin;
|
|
|
import org.apache.hadoop.hdds.HddsUtils;
|
|
|
import org.apache.hadoop.hdds.conf.OzoneConfiguration;
|
|
|
import org.apache.hadoop.hdds.protocol.DatanodeDetails;
|
|
|
import org.apache.hadoop.ozone.container.common.helpers.ContainerUtils;
|
|
|
import org.apache.hadoop.ozone.container.common.statemachine
|
|
|
.DatanodeStateMachine;
|
|
|
+import org.apache.hadoop.util.ServicePlugin;
|
|
|
+import org.apache.hadoop.util.StringUtils;
|
|
|
import org.slf4j.Logger;
|
|
|
import org.slf4j.LoggerFactory;
|
|
|
|
|
|
import java.io.File;
|
|
|
import java.io.IOException;
|
|
|
import java.net.InetAddress;
|
|
|
+import java.util.List;
|
|
|
import java.util.UUID;
|
|
|
|
|
|
+import static org.apache.hadoop.ozone.OzoneConfigKeys.HDDS_DATANODE_PLUGINS_KEY;
|
|
|
+import static org.apache.hadoop.util.ExitUtil.terminate;
|
|
|
+
|
|
|
/**
|
|
|
* Datanode service plugin to start the HDDS container services.
|
|
|
*/
|
|
|
-public class HddsDatanodeService implements DataNodeServicePlugin {
|
|
|
+public class HddsDatanodeService implements ServicePlugin {
|
|
|
|
|
|
private static final Logger LOG = LoggerFactory.getLogger(
|
|
|
HddsDatanodeService.class);
|
|
|
|
|
|
- private final boolean isOzoneEnabled;
|
|
|
|
|
|
private Configuration conf;
|
|
|
private DatanodeDetails datanodeDetails;
|
|
|
private DatanodeStateMachine datanodeStateMachine;
|
|
|
-
|
|
|
- public HddsDatanodeService() {
|
|
|
- try {
|
|
|
- OzoneConfiguration.activate();
|
|
|
- this.conf = new OzoneConfiguration();
|
|
|
- this.isOzoneEnabled = HddsUtils.isHddsEnabled(conf);
|
|
|
- if (isOzoneEnabled) {
|
|
|
- this.datanodeDetails = getDatanodeDetails(conf);
|
|
|
- String hostname = DataNode.getHostName(conf);
|
|
|
- String ip = InetAddress.getByName(hostname).getHostAddress();
|
|
|
- this.datanodeDetails.setHostName(hostname);
|
|
|
- this.datanodeDetails.setIpAddress(ip);
|
|
|
- }
|
|
|
- } catch (IOException e) {
|
|
|
- throw new RuntimeException("Can't start the HDDS datanode plugin", e);
|
|
|
- }
|
|
|
- }
|
|
|
+ private List<ServicePlugin> plugins;
|
|
|
|
|
|
@Override
|
|
|
public void start(Object service) {
|
|
|
- if (isOzoneEnabled) {
|
|
|
+ OzoneConfiguration.activate();
|
|
|
+ if (service instanceof Configurable) {
|
|
|
+ conf = new OzoneConfiguration(((Configurable) service).getConf());
|
|
|
+ } else {
|
|
|
+ conf = new OzoneConfiguration();
|
|
|
+ }
|
|
|
+ if (HddsUtils.isHddsEnabled(conf)) {
|
|
|
try {
|
|
|
- DataNode dataNode = (DataNode) service;
|
|
|
- datanodeDetails.setInfoPort(dataNode.getInfoPort());
|
|
|
- datanodeDetails.setInfoSecurePort(dataNode.getInfoSecurePort());
|
|
|
+ String hostname = DataNode.getHostName(conf);
|
|
|
+ String ip = InetAddress.getByName(hostname).getHostAddress();
|
|
|
+ datanodeDetails = initializeDatanodeDetails();
|
|
|
+ datanodeDetails.setHostName(hostname);
|
|
|
+ datanodeDetails.setIpAddress(ip);
|
|
|
+
|
|
|
+ //Below block should be removed as part of HDFS-13324
|
|
|
+ if (service != null) {
|
|
|
+ DataNode dataNode = (DataNode) service;
|
|
|
+ datanodeDetails.setInfoPort(dataNode.getInfoPort());
|
|
|
+ datanodeDetails.setInfoSecurePort(dataNode.getInfoSecurePort());
|
|
|
+ }
|
|
|
datanodeStateMachine = new DatanodeStateMachine(datanodeDetails, conf);
|
|
|
+ startPlugins();
|
|
|
+ // Starting HDDS Daemons
|
|
|
datanodeStateMachine.startDaemon();
|
|
|
} catch (IOException e) {
|
|
|
throw new RuntimeException("Can't start the HDDS datanode plugin", e);
|
|
@@ -84,11 +90,11 @@ public class HddsDatanodeService implements DataNodeServicePlugin {
|
|
|
}
|
|
|
|
|
|
/**
|
|
|
- * Returns ContainerNodeIDProto or null in case of Error.
|
|
|
+ * Returns DatanodeDetails or null in case of Error.
|
|
|
*
|
|
|
- * @return ContainerNodeIDProto
|
|
|
+ * @return DatanodeDetails
|
|
|
*/
|
|
|
- private static DatanodeDetails getDatanodeDetails(Configuration conf)
|
|
|
+ private DatanodeDetails initializeDatanodeDetails()
|
|
|
throws IOException {
|
|
|
String idFilePath = HddsUtils.getDatanodeIdFilePath(conf);
|
|
|
if (idFilePath == null || idFilePath.isEmpty()) {
|
|
@@ -111,24 +117,62 @@ public class HddsDatanodeService implements DataNodeServicePlugin {
|
|
|
return DatanodeDetails.newBuilder().setUuid(datanodeUuid).build();
|
|
|
}
|
|
|
}
|
|
|
+ private void startPlugins() {
|
|
|
+ try {
|
|
|
+ plugins = conf.getInstances(HDDS_DATANODE_PLUGINS_KEY,
|
|
|
+ ServicePlugin.class);
|
|
|
+ } catch (RuntimeException e) {
|
|
|
+ String pluginsValue = conf.get(HDDS_DATANODE_PLUGINS_KEY);
|
|
|
+ LOG.error("Unable to load HDDS DataNode plugins. " +
|
|
|
+ "Specified list of plugins: {}",
|
|
|
+ pluginsValue, e);
|
|
|
+ throw e;
|
|
|
+ }
|
|
|
+ for (ServicePlugin plugin : plugins) {
|
|
|
+ try {
|
|
|
+ plugin.start(this);
|
|
|
+ LOG.info("Started plug-in {}", plugin);
|
|
|
+ } catch (Throwable t) {
|
|
|
+ LOG.warn("ServicePlugin {} could not be started", plugin, t);
|
|
|
+ }
|
|
|
+ }
|
|
|
+ }
|
|
|
|
|
|
+ public Configuration getConf() {
|
|
|
+ return conf;
|
|
|
+ }
|
|
|
/**
|
|
|
*
|
|
|
* Return DatanodeDetails if set, return null otherwise.
|
|
|
*
|
|
|
* @return DatanodeDetails
|
|
|
*/
|
|
|
+ @VisibleForTesting
|
|
|
public DatanodeDetails getDatanodeDetails() {
|
|
|
return datanodeDetails;
|
|
|
}
|
|
|
|
|
|
- @InterfaceAudience.Private
|
|
|
+ @VisibleForTesting
|
|
|
public DatanodeStateMachine getDatanodeStateMachine() {
|
|
|
return datanodeStateMachine;
|
|
|
}
|
|
|
|
|
|
+ public void join() throws InterruptedException {
|
|
|
+ datanodeStateMachine.join();
|
|
|
+ }
|
|
|
+
|
|
|
@Override
|
|
|
public void stop() {
|
|
|
+ if (plugins != null) {
|
|
|
+ for (ServicePlugin plugin : plugins) {
|
|
|
+ try {
|
|
|
+ plugin.stop();
|
|
|
+ LOG.info("Stopped plug-in {}", plugin);
|
|
|
+ } catch (Throwable t) {
|
|
|
+ LOG.warn("ServicePlugin {} could not be stopped", plugin, t);
|
|
|
+ }
|
|
|
+ }
|
|
|
+ }
|
|
|
if (datanodeStateMachine != null) {
|
|
|
datanodeStateMachine.stopDaemon();
|
|
|
}
|
|
@@ -137,4 +181,20 @@ public class HddsDatanodeService implements DataNodeServicePlugin {
|
|
|
@Override
|
|
|
public void close() throws IOException {
|
|
|
}
|
|
|
+
|
|
|
+ public static HddsDatanodeService createHddsDatanodeService(String args[]) {
|
|
|
+ StringUtils.startupShutdownMessage(HddsDatanodeService.class, args, LOG);
|
|
|
+ return new HddsDatanodeService();
|
|
|
+ }
|
|
|
+
|
|
|
+ public static void main(String args[]) {
|
|
|
+ try {
|
|
|
+ HddsDatanodeService hddsDatanodeService = createHddsDatanodeService(args);
|
|
|
+ hddsDatanodeService.start(null);
|
|
|
+ hddsDatanodeService.join();
|
|
|
+ } catch (Throwable e) {
|
|
|
+ LOG.error("Exception in while starting HddsDatanodeService.", e);
|
|
|
+ terminate(1, e);
|
|
|
+ }
|
|
|
+ }
|
|
|
}
|