Skip to content
Merged
Show file tree
Hide file tree
Changes from 1 commit
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 @@ -39,6 +39,8 @@ public final class ContainerID implements Comparable<ContainerID> {
LongCodec.get(), ContainerID::valueOf, c -> c.id,
DelegatedCodec.CopyType.SHALLOW);

public static final ContainerID MIN = ContainerID.valueOf(0);

public static Codec<ContainerID> getCodec() {
return CODEC;
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -19,19 +19,18 @@

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;
import java.util.NavigableSet;
import java.util.Objects;
import java.util.Optional;
import java.util.Random;
import java.util.Set;
import java.util.concurrent.TimeoutException;
import java.util.concurrent.locks.Lock;
import java.util.concurrent.locks.ReentrantLock;
import java.util.stream.Collectors;

import com.google.common.annotations.VisibleForTesting;
import com.google.common.base.Preconditions;
Expand Down Expand Up @@ -144,55 +143,23 @@ public ContainerInfo getContainer(final ContainerID id)
public List<ContainerInfo> getContainers(final ContainerID startID,
final int count) {
scmContainerManagerMetrics.incNumListContainersOps();
final List<ContainerID> containersIds =
new ArrayList<>(containerStateManager.getContainerIDs());
Collections.sort(containersIds);
List<ContainerInfo> containers;
lock.lock();
try {
containers = containersIds.stream()
.filter(id -> id.compareTo(startID) >= 0).limit(count)
.map(containerStateManager::getContainer)
.collect(Collectors.toList());
} finally {
lock.unlock();
}
return containers;
return toContainers(filterSortAndLimit(startID, count,
containerStateManager.getContainerIDs()));
}

@Override
public List<ContainerInfo> getContainers(final LifeCycleState state) {
List<ContainerInfo> containers;
lock.lock();
try {
containers = containerStateManager.getContainerIDs(state).stream()
.map(containerStateManager::getContainer)
.filter(Objects::nonNull).collect(Collectors.toList());
} finally {
lock.unlock();
}
return containers;
scmContainerManagerMetrics.incNumListContainersOps();
return toContainers(containerStateManager.getContainerIDs(state));
}

@Override
public List<ContainerInfo> getContainers(final ContainerID startID,
final int count,
final LifeCycleState state) {
scmContainerManagerMetrics.incNumListContainersOps();
final List<ContainerID> containersIds =
new ArrayList<>(containerStateManager.getContainerIDs(state));
Collections.sort(containersIds);
List<ContainerInfo> containers;
lock.lock();
try {
containers = containersIds.stream()
.filter(id -> id.compareTo(startID) >= 0).limit(count)
.map(containerStateManager::getContainer)
.collect(Collectors.toList());
} finally {
lock.unlock();
}
return containers;
return toContainers(filterSortAndLimit(startID, count,
containerStateManager.getContainerIDs(state)));
}

@Override
Expand Down Expand Up @@ -431,18 +398,24 @@ public void notifyContainerReportProcessing(final boolean isFullReport,
public void deleteContainer(final ContainerID cid)
throws IOException, TimeoutException {
HddsProtos.ContainerID protoId = cid.getProtobuf();

final boolean found;
lock.lock();
try {
if (containerExist(cid)) {
found = containerExist(cid);
if (found) {
containerStateManager.removeContainer(protoId);
scmContainerManagerMetrics.incNumSuccessfulDeleteContainers();
} else {
scmContainerManagerMetrics.incNumFailureDeleteContainers();
throwContainerNotFoundException(cid);
}
} finally {
lock.unlock();
}

if (found) {
scmContainerManagerMetrics.incNumSuccessfulDeleteContainers();
} else {
scmContainerManagerMetrics.incNumFailureDeleteContainers();
throwContainerNotFoundException(cid);
}
}

@Override
Expand Down Expand Up @@ -471,4 +444,37 @@ public ContainerStateManager getContainerStateManager() {
public SCMHAManager getSCMHAManager() {
return haManager;
}

private static List<ContainerID> filterSortAndLimit(
Copy link
Contributor

@sodonnel sodonnel Jun 27, 2023

Choose a reason for hiding this comment

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

I think this could be made more efficient. The current algorithm takes the set, forms a new list of all elements. Then filters, sorted and returns the range.

Using an adaption of the priorityQueue technique here - https://www.baeldung.com/java-array-top-elements

We could avoid the copy from set to list, and the potentially large sort.

for (contaienrID cid : set) {
    if (id > startID)
        maxHeap.add(number);

        if (maxHeap.size() > count) {
            maxHeap.poll();
        }
    }
});
return new ArrayList<>(maxHeap);
// Might need reversed, I am not sure

I think this would be more memory efficient on large container sets, and should perform better than the full sort

Copy link
Contributor Author

@adoroszlai adoroszlai Jun 27, 2023

Choose a reason for hiding this comment

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

Thanks for the suggestion, I'll update the patch with it.

FYI there is a bug in that solution due to PriorityQueue#iterator():

The iterator does not return the elements in any particular order.

so new ArrayList<>(maxHeap) randomizes the result.

It seems to work for (small?) Integers, but not for any random Comparable.

Copy link
Contributor

Choose a reason for hiding this comment

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

I wonder how many people have not noticed that issue, and just implemented the linked solution as is!

ContainerID startID, int count, Set<ContainerID> set) {

List<ContainerID> ids = new ArrayList<>(set);

// only filter if startID is not the minimum one
if (!startID.equals(ContainerID.MIN)) {
ids.removeIf(id -> id.compareTo(startID) < 0);
}

// sort after removing unnecessary items
Collections.sort(ids);

// limit
return ids.subList(0, Math.min(ids.size(), count));
}

/**
* 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 @@ -18,6 +18,7 @@
package org.apache.hadoop.hdds.scm.container.states;

import com.google.common.base.Preconditions;
import com.google.common.collect.ImmutableSortedSet;
import org.apache.hadoop.hdds.scm.container.ContainerID;
import org.apache.hadoop.hdds.scm.exceptions.SCMException;
import org.slf4j.Logger;
Expand Down Expand Up @@ -189,7 +190,7 @@ public NavigableSet<ContainerID> getCollection(T key) {
Preconditions.checkNotNull(key);

if (this.attributeMap.containsKey(key)) {
return Collections.unmodifiableNavigableSet(this.attributeMap.get(key));
return ImmutableSortedSet.copyOf(this.attributeMap.get(key));
}
if (LOG.isDebugEnabled()) {
LOG.debug("No such Key. Key {}", key);
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -28,6 +28,7 @@

import com.google.common.base.Preconditions;

import com.google.common.collect.ImmutableSet;
import org.apache.hadoop.hdds.client.ReplicationConfig;
import org.apache.hadoop.hdds.scm.container.ContainerID;
import org.apache.hadoop.hdds.scm.container.ContainerReplica;
Expand Down Expand Up @@ -295,7 +296,7 @@ public void updateState(ContainerID containerID, LifeCycleState currentState,
}

public Set<ContainerID> getAllContainerIDs() {
return Collections.unmodifiableSet(containerMap.keySet());
return ImmutableSet.copyOf(containerMap.keySet());
}

/**
Expand Down