Skip to content
Merged
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 @@ -17,16 +17,11 @@

package org.apache.hadoop.hdds.scm.container;

import static java.util.Comparator.reverseOrder;
import static org.apache.hadoop.hdds.scm.ha.SequenceIdGenerator.CONTAINER_ID;
import static org.apache.hadoop.hdds.utils.CollectionUtils.findTopN;

import com.google.common.annotations.VisibleForTesting;
import com.google.common.base.Preconditions;
import java.io.IOException;
import java.util.ArrayList;
import java.util.Collection;
import java.util.Collections;
import java.util.Iterator;
import java.util.List;
import java.util.Map;
Expand Down Expand Up @@ -144,35 +139,33 @@ public ContainerInfo getContainer(final ContainerID id)

@Override
public List<ContainerInfo> getContainers(ReplicationType type) {
return toContainers(containerStateManager.getContainerIDs(type));
return containerStateManager.getContainerInfos(type);
}

@Override
public List<ContainerInfo> getContainers(final ContainerID startID,
final int count) {
scmContainerManagerMetrics.incNumListContainersOps();
return toContainers(filterSortAndLimit(startID, count,
containerStateManager.getContainerIDs()));
return containerStateManager.getContainerInfos(startID, count);
}

@Override
public List<ContainerInfo> getContainers(final LifeCycleState state) {
scmContainerManagerMetrics.incNumListContainersOps();
return toContainers(containerStateManager.getContainerIDs(state));
return containerStateManager.getContainerInfos(state);
}

@Override
public List<ContainerInfo> getContainers(final ContainerID startID,
final int count,
final LifeCycleState state) {
scmContainerManagerMetrics.incNumListContainersOps();
return toContainers(filterSortAndLimit(startID, count,
containerStateManager.getContainerIDs(state)));
return containerStateManager.getContainerInfos(state, startID, count);
}

@Override
public int getContainerStateCount(final LifeCycleState state) {
return containerStateManager.getContainerIDs(state).size();
return containerStateManager.getContainerCount(state);
}

@Override
Expand Down Expand Up @@ -469,33 +462,4 @@ public ContainerStateManager getContainerStateManager() {
public SCMHAManager getSCMHAManager() {
return haManager;
}

private static List<ContainerID> filterSortAndLimit(
ContainerID startID, int count, Set<ContainerID> set) {

if (ContainerID.MIN.equals(startID) && count >= set.size()) {
List<ContainerID> list = new ArrayList<>(set);
Collections.sort(list);
return list;
}

return findTopN(set, count, reverseOrder(),
id -> id.compareTo(startID) >= 0);
}

/**
* Returns a list of all containers identified by {@code ids}.
*/
private List<ContainerInfo> toContainers(Collection<ContainerID> ids) {
List<ContainerInfo> containers = new ArrayList<>(ids.size());

for (ContainerID id : ids) {
ContainerInfo container = containerStateManager.getContainer(id);
if (container != null) {
containers.add(container);
}
}

return containers;
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -54,6 +54,10 @@ private ContainerReplica(ContainerReplicaBuilder b) {
sequenceId = b.sequenceId;
}

public ContainerID getContainerID() {
return containerID;
}

/**
* Returns the DatanodeDetails to which this replica belongs.
*
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,6 +18,7 @@
package org.apache.hadoop.hdds.scm.container;

import java.io.IOException;
import java.util.List;
import java.util.Map;
import java.util.NavigableSet;
import java.util.Set;
Expand Down Expand Up @@ -103,22 +104,33 @@ public interface ContainerStateManager {
boolean contains(ContainerID containerID);

/**
* Returns the ID of all the managed containers.
* Get {@link ContainerInfo}s.
*
* @return Set of {@link ContainerID}
* @param start the start {@link ContainerID} (inclusive)
* @param count the size limit
* @return a list of {@link ContainerInfo};
*/
Set<ContainerID> getContainerIDs();
List<ContainerInfo> getContainerInfos(ContainerID start, int count);

/**
* Get {@link ContainerInfo}s for the given state.
*
* @param start the start {@link ContainerID} (inclusive)
* @param count the size limit
* @return a list of {@link ContainerInfo};
*/
Set<ContainerID> getContainerIDs(LifeCycleState state);
List<ContainerInfo> getContainerInfos(LifeCycleState state, ContainerID start, int count);

/** @return all {@link ContainerInfo}s for the given state. */
List<ContainerInfo> getContainerInfos(LifeCycleState state);

/**
* Returns the IDs of the Containers whose ReplicationType matches the given type.
* @return number of containers for the given state.
*/
Set<ContainerID> getContainerIDs(ReplicationType type);
int getContainerCount(LifeCycleState state);

/** @return all {@link ContainerInfo}s for the given type. */
List<ContainerInfo> getContainerInfos(ReplicationType type);

/**
*
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -32,13 +32,14 @@
import static org.apache.hadoop.hdds.scm.ScmConfigKeys.OZONE_SCM_CONTAINER_LOCK_STRIPE_SIZE;
import static org.apache.hadoop.hdds.scm.ScmConfigKeys.OZONE_SCM_CONTAINER_LOCK_STRIPE_SIZE_DEFAULT;

import com.google.common.base.Preconditions;
import com.google.common.util.concurrent.Striped;
import java.io.IOException;
import java.util.EnumMap;
import java.util.HashSet;
import java.util.List;
import java.util.Map;
import java.util.NavigableSet;
import java.util.Objects;
import java.util.Set;
import java.util.concurrent.ConcurrentHashMap;
import java.util.concurrent.locks.ReadWriteLock;
Expand Down Expand Up @@ -68,6 +69,7 @@
import org.apache.hadoop.ozone.common.statemachine.InvalidStateTransitionException;
import org.apache.hadoop.ozone.common.statemachine.StateMachine;
import org.apache.ratis.util.AutoCloseableLock;
import org.apache.ratis.util.Preconditions;
import org.apache.ratis.util.function.CheckedConsumer;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
Expand Down Expand Up @@ -239,7 +241,7 @@ private void initialize() throws IOException {

while (iterator.hasNext()) {
final ContainerInfo container = iterator.next().getValue();
Preconditions.checkNotNull(container);
Objects.requireNonNull(container, "container == null");
containers.addContainer(container);
if (container.getState() == LifeCycleState.OPEN) {
try {
Expand Down Expand Up @@ -268,23 +270,37 @@ private void initialize() throws IOException {
}

@Override
public Set<ContainerID> getContainerIDs() {
public List<ContainerInfo> getContainerInfos(ContainerID start, int count) {
try (AutoCloseableLock ignored = readLock()) {
return containers.getAllContainerIDs();
return containers.getContainerInfos(start, count);
}
}

@Override
public Set<ContainerID> getContainerIDs(final LifeCycleState state) {
public List<ContainerInfo> getContainerInfos(LifeCycleState state, ContainerID start, int count) {
try (AutoCloseableLock ignored = readLock()) {
return containers.getContainerIDsByState(state);
return containers.getContainerInfos(state, start, count);
}
}

@Override
public Set<ContainerID> getContainerIDs(final ReplicationType type) {
public List<ContainerInfo> getContainerInfos(final LifeCycleState state) {
try (AutoCloseableLock ignored = readLock()) {
return containers.getContainerIDsByType(type);
return containers.getContainerInfos(state);
}
}

@Override
public List<ContainerInfo> getContainerInfos(ReplicationType type) {
try (AutoCloseableLock ignored = readLock()) {
return containers.getContainerInfos(type);
}
}

@Override
public int getContainerCount(final LifeCycleState state) {
try (AutoCloseableLock ignored = readLock()) {
return containers.getContainerCount(state);
}
}

Expand All @@ -303,7 +319,7 @@ public void addContainer(final ContainerInfoProto containerInfo)
// ClosedPipelineException once ClosedPipelineException is introduced
// in PipelineManager.

Preconditions.checkNotNull(containerInfo);
Objects.requireNonNull(containerInfo, "containerInfo == null");
final ContainerInfo container = ContainerInfo.fromProtobuf(containerInfo);
final ContainerID containerID = container.containerID();
final PipelineID pipelineID = container.getPipelineID();
Expand Down Expand Up @@ -403,7 +419,7 @@ public Set<ContainerReplica> getContainerReplicas(final ContainerID id) {
public void updateContainerReplica(final ContainerID id,
final ContainerReplica replica) {
try (AutoCloseableLock ignored = writeLock(id)) {
containers.updateContainerReplica(id, replica);
containers.updateContainerReplica(replica);
// Clear any pending additions for this replica as we have now seen it.
containerReplicaPendingOps.completeAddReplica(id,
replica.getDatanodeDetails(), replica.getReplicaIndex());
Expand All @@ -413,8 +429,10 @@ public void updateContainerReplica(final ContainerID id,
@Override
public void removeContainerReplica(final ContainerID id,
final ContainerReplica replica) {
//TODO remove ContainerID parameter
Copy link
Contributor Author

Choose a reason for hiding this comment

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

Filed HDDS-12572

Preconditions.assertEquals(id, replica.getContainerID(), "containerID");
try (AutoCloseableLock ignored = writeLock(id)) {
containers.removeContainerReplica(id, replica);
containers.removeContainerReplica(id, replica.getDatanodeDetails().getID());
// Remove any pending delete replication operations for the deleted
// replica.
containerReplicaPendingOps.completeDeleteReplica(id,
Expand Down Expand Up @@ -601,9 +619,9 @@ public Builder setContainerReplicaPendingOps(
}

public ContainerStateManager build() throws IOException {
Preconditions.checkNotNull(conf);
Preconditions.checkNotNull(pipelineMgr);
Preconditions.checkNotNull(table);
Objects.requireNonNull(conf, "conf == null");
Objects.requireNonNull(pipelineMgr, "pipelineMgr == null");
Objects.requireNonNull(table, "table == null");

final ContainerStateManager csm = new ContainerStateManagerImpl(
conf, pipelineMgr, table, transactionBuffer,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -25,6 +25,7 @@
import java.util.EnumMap;
import java.util.NavigableSet;
import java.util.Objects;
import java.util.SortedSet;
import java.util.TreeSet;
import org.apache.hadoop.hdds.scm.container.ContainerID;
import org.apache.hadoop.hdds.scm.exceptions.SCMException;
Expand Down Expand Up @@ -138,6 +139,15 @@ public NavigableSet<ContainerID> getCollection(T key) {
return ImmutableSortedSet.copyOf(get(key));
}

public SortedSet<ContainerID> tailSet(T key, ContainerID start) {
Objects.requireNonNull(start, "start == null");
return get(key).tailSet(start);
}

public int count(T key) {
return get(key).size();
}

/**
* Moves a ContainerID from one bucket to another.
*
Expand Down
Loading