Skip to content
Closed
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -120,6 +120,11 @@ public abstract Container importContainer(
TarContainerPacker packer)
throws IOException;

/**
* Stop the Handler.
*/
public abstract void stop();

/**
* Marks the container for closing. Moves the container to CLOSING state.
*
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -23,17 +23,22 @@
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.ozone.container.common.statemachine.StateContext;
import org.apache.hadoop.util.concurrent.HadoopExecutors;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

import java.util.ArrayList;
import java.util.List;
import java.util.concurrent.ScheduledExecutorService;
import java.util.concurrent.TimeUnit;

/**
* ReportManager is responsible for managing all the {@link ReportPublisher}
* and also provides {@link ScheduledExecutorService} to ReportPublisher
* which should be used for scheduling the reports.
*/
public final class ReportManager {
private static final Logger LOG =
LoggerFactory.getLogger(ReportManager.class);

private final StateContext context;
private final List<ReportPublisher> publishers;
Expand Down Expand Up @@ -71,6 +76,11 @@ public void init() {
*/
public void shutdown() {
executorService.shutdown();
try {
executorService.awaitTermination(1, TimeUnit.DAYS);
} catch (Exception e) {
LOG.error("failed to shutdown Report Manager", e);
Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

The first letter of the first word as a capital letter maybe better.

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Do you want timeUnit to be Days?

}
}

/**
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -57,6 +57,7 @@
import java.util.Collections;
import java.util.List;
import java.util.UUID;
import java.util.concurrent.TimeUnit;

/**
* Creates a Grpc server endpoint that acts as the communication layer for
Expand Down Expand Up @@ -172,6 +173,11 @@ public void start() throws IOException {
public void stop() {
if (isStarted) {
server.shutdown();
try {
server.awaitTermination(1, TimeUnit.DAYS);
Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

It's better to read the value from configuration file.

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Same question as above.

} catch (Exception e) {
LOG.error("failed to shutdown XceiverServerGrpc", e);
}
isStarted = false;
}
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -158,6 +158,11 @@ public VolumeChoosingPolicy getVolumeChoosingPolicyForTesting() {
return volumeChoosingPolicy;
}

@Override
public void stop() {
blockDeletingService.shutdown();
}

@Override
public ContainerCommandResponseProto handle(
ContainerCommandRequestProto request, Container container,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -200,6 +200,7 @@ public void stop() {
stopContainerScrub();
writeChannel.stop();
readChannel.stop();
this.handlers.values().forEach(Handler::stop);
hddsDispatcher.shutdown();
volumeSet.shutdown();
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -184,6 +184,7 @@ public NodeState getNodeState(DatanodeDetails datanodeDetails) {
public void close() throws IOException {
unregisterMXBean();
metrics.unRegister();
nodeStateManager.close();
}

/**
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -83,6 +83,7 @@
import org.apache.hadoop.hdfs.DFSUtil;
import org.apache.hadoop.io.IOUtils;
import org.apache.hadoop.ipc.RPC;
import org.apache.hadoop.metrics2.MetricsSystem;
import org.apache.hadoop.metrics2.lib.DefaultMetricsSystem;
import org.apache.hadoop.metrics2.util.MBeans;
import org.apache.hadoop.ozone.OzoneConfigKeys;
Expand Down Expand Up @@ -205,6 +206,7 @@ public final class StorageContainerManager extends ServiceRuntimeInfoImpl
private final OzoneConfiguration configuration;
private final SafeModeHandler safeModeHandler;
private SCMContainerMetrics scmContainerMetrics;
private MetricsSystem ms;
Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Suggest modify the instance variable name, so that it readability.


/**
* Creates a new StorageContainerManager. Configuration will be
Expand Down Expand Up @@ -898,7 +900,7 @@ public void start() throws IOException {
buildRpcServerStartMessage(
"StorageContainerLocationProtocol RPC server",
getClientRpcAddress()));
DefaultMetricsSystem.initialize("StorageContainerManager");
ms = DefaultMetricsSystem.initialize("StorageContainerManager");

commandWatcherLeaseManager.start();
getClientProtocolServer().start();
Expand Down Expand Up @@ -993,6 +995,10 @@ public void stop() {
metrics.unRegister();
}

if (ms != null) {
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

in order of precedence, i think closing scmMetadataStore takes more precedence than Metrics. Shall we move this close statement to end.

ms.stop();
}

unregisterMXBean();
if (scmContainerMetrics != null) {
scmContainerMetrics.unRegister();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -203,7 +203,7 @@ void initializeConfiguration() throws IOException {
1, TimeUnit.SECONDS);
conf.setTimeDuration(HddsConfigKeys.HDDS_HEARTBEAT_INTERVAL, 1,
TimeUnit.SECONDS);
conf.setInt(OzoneConfigKeys.OZONE_CONTAINER_CACHE_SIZE, 8);
conf.setInt(OzoneConfigKeys.OZONE_CONTAINER_CACHE_SIZE, 2);
Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

What's the purpose of change the cache size?

}

@Override
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -340,19 +340,20 @@ public void stop() {
ozoneManager.join();
}

if (!hddsDatanodes.isEmpty()) {
LOG.info("Shutting the HddsDatanodes");
hddsDatanodes.parallelStream()
.forEach(dn -> {
dn.stop();
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

shall we wrap them in try catch to cleanup all of them silently.

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Thanks for the review Ajay, I didnot understand this comment completely. Can you please elaborate. ?

dn.join();
});
}

if (scm != null) {
LOG.info("Stopping the StorageContainerManager");
scm.stop();
scm.join();
}

if (!hddsDatanodes.isEmpty()) {
LOG.info("Shutting the HddsDatanodes");
for (HddsDatanodeService hddsDatanode : hddsDatanodes) {
hddsDatanode.stop();
hddsDatanode.join();
}
}
}

/**
Expand Down Expand Up @@ -568,6 +569,8 @@ private void configureSCM() {
conf.set(ScmConfigKeys.OZONE_SCM_DATANODE_ADDRESS_KEY, "127.0.0.1:0");
conf.set(ScmConfigKeys.OZONE_SCM_HTTP_ADDRESS_KEY, "127.0.0.1:0");
conf.setInt(ScmConfigKeys.OZONE_SCM_HANDLER_COUNT_KEY, numOfScmHandlers);
conf.set(HddsConfigKeys.HDDS_SCM_WAIT_TIME_AFTER_SAFE_MODE_EXIT,
"3s");
configureSCMheartbeat();
}

Expand Down