You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@lucene.apache.org by GitBox <gi...@apache.org> on 2020/09/02 17:56:49 UTC

[GitHub] [lucene-solr] sigram commented on a change in pull request #1684: SOLR-14613: strongly typed placement plugin interface and implementation

sigram commented on a change in pull request #1684:
URL: https://github.com/apache/lucene-solr/pull/1684#discussion_r482181180



##########
File path: solr/core/src/java/org/apache/solr/cluster/placement/Node.java
##########
@@ -0,0 +1,25 @@
+/*
+ * 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.solr.cluster.placement;
+
+/**
+ * Representation of a SolrCloud node or server in the SolrCloud cluster.
+ */
+public interface Node extends PropertyValueSource {
+  String getNodeName();

Review comment:
       `getName`? we already know it's a node.

##########
File path: solr/core/src/java/org/apache/solr/cluster/placement/Cluster.java
##########
@@ -0,0 +1,53 @@
+/*
+ * 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.solr.cluster.placement;

Review comment:
       Eventually we can put it in `org.apache.solr.cluster` (maybe replace `SolrCluster`).

##########
File path: solr/core/src/java/org/apache/solr/cluster/placement/AddReplicasPlacementRequest.java
##########
@@ -0,0 +1,64 @@
+/*
+ * 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.solr.cluster.placement;
+
+import java.util.Set;
+
+/**
+ * <p>Request passed by Solr to a {@link PlacementPlugin} to compute placement for one or more {@link Replica}'s for one
+ * or more {@link Shard}'s of an existing {@link SolrCollection}.
+ * The shard might or might not already exist, plugin code can easily find out by using {@link SolrCollection#getShards()}
+ * and verifying if the shard name(s) from {@link #getShardNames()} are there.
+ *
+ * <p>The set of {@link Node}s on which the replicas should be placed
+ * is specified (defaults to being equal to the set returned by {@link Cluster#getLiveNodes()}).
+ */
+public interface AddReplicasPlacementRequest extends PlacementRequest {
+  /**
+   * The {@link SolrCollection} to add {@link Replica}(s) to.
+   */
+  SolrCollection getCollection();
+
+  /**
+   * <p>Shard name(s) for which new replicas placement should be computed. The shard(s) might exist or not (that's why this
+   * method returns a {@link Set} of {@link String}'s and not directly a set of {@link Shard} instances).
+   *
+   * <p>Note the Collection API allows specifying the shard name or a {@code _route_} parameter. The Solr implementation will
+   * convert either specification into the relevant shard name so the plugin code doesn't have to worry about this.
+   */
+  Set<String> getShardNames();
+
+  /**
+   * <p>Replicas should only be placed on nodes in the set returned by this method.
+   *
+   * <p>When Collection API calls do not specify a specific set of target nodes, replicas can be placed on any live node of
+   * the cluster. In such cases, this set will be equal to the set of all live nodes. The plugin placement code does not
+   * need to worry (or care) if a set of nodes was explicitly specified or not.
+   *
+   * @return never {@code null} and never empty set (if that set was to be empty for any reason, no placement would be
+   * possible and the Solr infrastructure driving the plugin code would detect the error itself rather than calling the plugin).
+   */
+  Set<Node> getTargetNodes();

Review comment:
       I'm concerned about the memory impact here ... consider a cluster of 10,000 nodes :) Maybe it's enough to return the set of node names (strings)? Also, a common scenario is that the requestor doesn't care about specific nodes, so defining that "empty set == all live nodes" makes more sense to me...

##########
File path: solr/core/src/java/org/apache/solr/cluster/placement/impl/AbstractPropertyValue.java
##########
@@ -0,0 +1,155 @@
+/*
+ * 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.solr.cluster.placement.impl;
+
+import org.apache.solr.cluster.placement.PropertyKey;
+import org.apache.solr.cluster.placement.PropertyValue;
+
+/**
+ * This class and static subclasses implement all subtypes of {@link PropertyValue}
+ */
+public abstract class AbstractPropertyValue implements PropertyValue {

Review comment:
       See my other comments about simplifying this hierarchy at the cost of relaxing the strong-typing here.

##########
File path: solr/core/src/java/org/apache/solr/cluster/placement/PropertyValue.java
##########
@@ -0,0 +1,144 @@
+/*
+ * 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.solr.cluster.placement;
+
+/**
+ *  <p>The value corresponding to a specific {@link PropertyKey}, in a specific context (e.g. property of a specific
+ *  {@link Node} instance). The context is tracked in the {@link PropertyKey} using a {@link PropertyValueSource}.
+ *
+ *  <p>Instances are obtained by first getting a key using {@link PropertyKeyFactory} then getting the corresponding
+ *  {@link PropertyValue} using {@link PropertyValueFetcher}.
+ */
+public interface PropertyValue {
+  /**
+   * The property key used for retrieving this property value.
+   */
+  PropertyKey getKey();
+
+  /**
+   *  Instances are obtained by first getting a key using {@link PropertyKeyFactory#createCoreCountKey} then calling
+   *  {@link PropertyValueFetcher#fetchProperties}, retrieving the appropriate {@link PropertyValue} from the returned map
+   *  using the {@link PropertyKey} as key and finally casting it to {@link PropertyValue.CoresCount}.
+   */
+  interface CoresCount extends PropertyValue {
+    /**
+     * Returns the number of cores on the {@link Node}) this instance was obtained from (i.e. instance
+     * passed to {@link PropertyKeyFactory#createCoreCountKey(Node)}).
+     */
+    int getCoresCount();
+  }
+
+  /**
+   *  Instances are obtained by first getting a key using {@link PropertyKeyFactory#createDiskTypeKey} then getting the
+   *  {@link org.apache.solr.cluster.placement.PropertyValue.DiskType} using {@link PropertyValueFetcher#fetchProperties} and retrieving (then casting) the
+   *  appropriate {@link PropertyValue} from the returned map using the {@link PropertyKey} as key.
+   */
+  interface DiskType extends PropertyValue {
+    /**
+     * Type of storage hardware used for the partition on which cores are stored on the {@link Node}) from which this instance
+     * was obtained (i.e. instance passed to {@link PropertyKeyFactory#createDiskTypeKey(Node)}).
+     */
+    HardwareType getHardwareType();
+
+    enum HardwareType {
+      SSD, ROTATIONAL, UNKNOWN
+    }
+  }
+
+  /**
+   *  Instances are obtained by first getting a key using {@link PropertyKeyFactory#createTotalDiskKey(Node)} (Node)} then getting the
+   *  {@link org.apache.solr.cluster.placement.PropertyValue.TotalDisk} using {@link PropertyValueFetcher#fetchProperties} and retrieving (then casting) the
+   *  appropriate {@link PropertyValue} fetched from the returned map using the {@link PropertyKey} as key.
+   */
+  interface TotalDisk extends PropertyValue {
+    /**
+     * Total disk size of the partition on which cores are stored on the {@link Node}) from which this instance was obtained
+     * (i.e. instance passed to {@link PropertyKeyFactory#createTotalDiskKey(Node)}).
+     */
+    long getTotalSizeGB();
+  }
+
+  /**
+   *  Instances are obtained by first getting a key using {@link PropertyKeyFactory#createFreeDiskKey(Node)} then getting the
+   *  {@link org.apache.solr.cluster.placement.PropertyValue.FreeDisk} using {@link PropertyValueFetcher#fetchProperties} and retrieving (then casting) the
+   *  appropriate {@link PropertyValue} fetched from the returned map using the {@link PropertyKey} as key.
+   */
+  interface FreeDisk extends PropertyValue {
+    /**
+     * Free disk size of the partition on which cores are stored on the {@link Node}) from which this instance was obtained
+     *  (i.e. instance passed to {@link PropertyKeyFactory#createDiskTypeKey(Node)}).
+     */
+    long getFreeSizeGB();
+  }
+
+
+  /**
+   *  Instances are obtained by first getting a key using {@link PropertyKeyFactory#createHeapUsageKey(Node)} then calling
+   *  {@link PropertyValueFetcher#fetchProperties}, retrieving the appropriate {@link PropertyValue} from the returned map
+   *  using the {@link PropertyKey} as key and finally casting it to {@link org.apache.solr.cluster.placement.PropertyValue.HeapUsage}.
+   */
+  interface HeapUsage extends PropertyValue {
+
+    /**
+     * Percentage between 0 and 100 of used heap over max heap.
+     */
+    double getUsedHeapMemoryUsage();
+  }
+
+  /**
+   * A {@link PropertyValue} representing a metric on the target {@link PropertyValueSource}.
+   *
+   * <p>Instances are obtained by first getting a key using {@link PropertyKeyFactory#createMetricKey(PropertyValueSource, String)}
+   * or {@link PropertyKeyFactory#createMetricKey(Node, String, PropertyKeyFactory.NodeMetricRegistry)} then calling
+   * {@link PropertyValueFetcher#fetchProperties}, retrieving the appropriate {@link PropertyValue} from the returned map
+   * using the {@link PropertyKey} as key and finally casting it to {@link org.apache.solr.cluster.placement.PropertyValue.Metric}.
+   */
+  interface Metric extends PropertyValue {
+    /**
+     * Returns the metric value from the {@link PropertyValueSource} from which it was retrieved.
+     */
+    Double getNumberValue();
+  }
+
+  /**
+   * A {@link PropertyValue} representing a sysprop (or System property) on the target {@link Node}.
+   *
+   *  Instances are obtained by first getting a key using {@link PropertyKeyFactory#createSyspropKey} then calling
+   *  {@link PropertyValueFetcher#fetchProperties}, retrieving the appropriate {@link PropertyValue} from the returned map
+   *  using the {@link PropertyKey} as key and finally casting it to {@link org.apache.solr.cluster.placement.PropertyValue.Sysprop}.
+   */
+  interface Sysprop extends PropertyValue {
+    /**

Review comment:
       +1.

##########
File path: solr/core/src/java/org/apache/solr/cluster/placement/Replica.java
##########
@@ -0,0 +1,46 @@
+/*
+ * 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.solr.cluster.placement;
+
+/**
+ * An instantiation (or one of the copies) of a given {@link Shard} of a given {@link SolrCollection}.
+ * Objects of this type are returned by the Solr framework to the plugin, they are not directly built by the plugin. When the
+ * plugin wants to add a replica it goes through appropriate method in {@link PlacementPlanFactory}).
+ */
+public interface Replica extends PropertyValueSource {
+  Shard getShard();
+
+  ReplicaType getType();
+  ReplicaState getState();

Review comment:
       SOLR-14680 doesn't include these, but includes `isLeader` (and some other props).

##########
File path: solr/core/src/java/org/apache/solr/cluster/placement/Replica.java
##########
@@ -0,0 +1,46 @@
+/*
+ * 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.solr.cluster.placement;
+
+/**
+ * An instantiation (or one of the copies) of a given {@link Shard} of a given {@link SolrCollection}.
+ * Objects of this type are returned by the Solr framework to the plugin, they are not directly built by the plugin. When the
+ * plugin wants to add a replica it goes through appropriate method in {@link PlacementPlanFactory}).
+ */
+public interface Replica extends PropertyValueSource {
+  Shard getShard();
+
+  ReplicaType getType();
+  ReplicaState getState();
+
+  String getReplicaName();
+  String getCoreName();
+
+  /**
+   * {@link Node} on which this {@link Replica} is located.
+   */
+  Node getNode();

Review comment:
       Do we need a Node instance or just a node name here?

##########
File path: solr/core/src/java/org/apache/solr/cluster/placement/PropertyValue.java
##########
@@ -0,0 +1,144 @@
+/*
+ * 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.solr.cluster.placement;
+
+/**
+ *  <p>The value corresponding to a specific {@link PropertyKey}, in a specific context (e.g. property of a specific
+ *  {@link Node} instance). The context is tracked in the {@link PropertyKey} using a {@link PropertyValueSource}.
+ *
+ *  <p>Instances are obtained by first getting a key using {@link PropertyKeyFactory} then getting the corresponding
+ *  {@link PropertyValue} using {@link PropertyValueFetcher}.
+ */
+public interface PropertyValue {
+  /**
+   * The property key used for retrieving this property value.
+   */
+  PropertyKey getKey();
+
+  /**
+   *  Instances are obtained by first getting a key using {@link PropertyKeyFactory#createCoreCountKey} then calling
+   *  {@link PropertyValueFetcher#fetchProperties}, retrieving the appropriate {@link PropertyValue} from the returned map
+   *  using the {@link PropertyKey} as key and finally casting it to {@link PropertyValue.CoresCount}.
+   */
+  interface CoresCount extends PropertyValue {

Review comment:
       I see the same problem here as with `PropertyKey`. The API surface here becomes very large.
   
   Also, in practice users of this API will always have to do an `instanceof` check and then cast `PropertyValue` to one of its subclasses. Is this significantly better than retrieving a predefined key from a Map and casting an Object to Integer? I'm not so sure, but I'm sure it would be much easier to use.

##########
File path: solr/core/src/java/org/apache/solr/cluster/placement/impl/PluginInteractionsImpl.java
##########
@@ -0,0 +1,190 @@
+/*

Review comment:
       That's frankly a very confusing name and there's little reason for grouping these classes here. Why not put all implementation classes separately into an *.impl package?
   
   (I know, this increases the number of files :) but that's directly the function of the number of defined interfaces.)

##########
File path: solr/core/src/java/org/apache/solr/cluster/placement/PlacementPluginConfig.java
##########
@@ -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.solr.cluster.placement;
+
+/**
+ * <p>Configuration passed by Solr to {@link PlacementPluginFactory#createPluginInstance(PlacementPluginConfig)} so that plugin instances
+ * ({@link PlacementPlugin}) created by the factory can easily retrieve their configuration.
+ */
+public interface PlacementPluginConfig {

Review comment:
       This in fact could be a general-purpose interface to many other configs that are based on key/value properties. See eg. `ZkNodeProps`.

##########
File path: solr/core/src/java/org/apache/solr/cluster/placement/plugins/SamplePluginMinimizeCores.java
##########
@@ -0,0 +1,132 @@
+/*
+ * 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.solr.cluster.placement.plugins;
+
+import java.util.ArrayList;
+import java.util.Collections;
+import java.util.Comparator;
+import java.util.HashSet;
+import java.util.Iterator;
+import java.util.Set;
+import java.util.Map;
+
+import com.google.common.collect.Ordering;
+import com.google.common.collect.TreeMultimap;
+import org.apache.solr.cluster.placement.Cluster;
+import org.apache.solr.cluster.placement.CoresCountPropertyValue;
+import org.apache.solr.cluster.placement.CreateNewCollectionPlacementRequest;
+import org.apache.solr.cluster.placement.Node;
+import org.apache.solr.cluster.placement.PlacementException;
+import org.apache.solr.cluster.placement.PlacementPlugin;
+import org.apache.solr.cluster.placement.PropertyKey;
+import org.apache.solr.cluster.placement.PropertyKeyFactory;
+import org.apache.solr.cluster.placement.PropertyValue;
+import org.apache.solr.cluster.placement.PropertyValueFetcher;
+import org.apache.solr.cluster.placement.Replica;
+import org.apache.solr.cluster.placement.ReplicaPlacement;
+import org.apache.solr.cluster.placement.PlacementRequest;
+import org.apache.solr.cluster.placement.PlacementPlan;
+import org.apache.solr.cluster.placement.PlacementPlanFactory;
+import org.apache.solr.common.util.SuppressForbidden;
+
+/**
+ * Implements placing replicas to minimize number of cores per {@link Node}, while not placing two replicas of the same
+ * shard on the same node.
+ *
+ * TODO: code not tested and never run, there are no implementation yet for used interfaces
+ */
+public class SamplePluginMinimizeCores implements PlacementPlugin {
+
+  @SuppressForbidden(reason = "Ordering.arbitrary() has no equivalent in Comparator class. Rather reuse than copy.")
+  public PlacementPlan computePlacement(Cluster cluster, PlacementRequest placementRequest, PropertyKeyFactory propertyFactory,
+                                        PropertyValueFetcher propertyFetcher, PlacementPlanFactory placementPlanFactory) throws PlacementException {
+    // This plugin only supports Creating a collection.
+    if (!(placementRequest instanceof CreateNewCollectionPlacementRequest)) {
+      throw new PlacementException("This toy plugin only supports creating collections");
+    }
+
+    final CreateNewCollectionPlacementRequest reqCreateCollection = (CreateNewCollectionPlacementRequest) placementRequest;
+
+    final int totalReplicasPerShard = reqCreateCollection.getNrtReplicationFactor() +
+        reqCreateCollection.getTlogReplicationFactor() + reqCreateCollection.getPullReplicationFactor();
+
+    if (cluster.getLiveNodes().size() < totalReplicasPerShard) {
+      throw new PlacementException("Cluster size too small for number of replicas per shard");
+    }
+
+    // Get number of cores on each Node
+    TreeMultimap<Integer, Node> nodesByCores = TreeMultimap.create(Comparator.naturalOrder(), Ordering.arbitrary());

Review comment:
       I agree with @murblanc about the need for efficient multi-fetching. But I also agree with @noblepaul :) that we could partially relax strong-typing requirements here in order to drastically reduce the API surface.

##########
File path: solr/core/src/java/org/apache/solr/cluster/placement/SolrCollection.java
##########
@@ -0,0 +1,66 @@
+/*
+ * 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.solr.cluster.placement;
+
+import java.util.Map;
+
+/**
+ * Represents a Collection in SolrCloud (unrelated to {@link java.util.Collection} that uses the nicer name).
+ */
+public interface SolrCollection {
+  /**
+   * The collection name (value passed to {@link Cluster#getCollection(String)}).
+   */
+  String getName();
+
+  /**
+   * <p>The {@link Shard}'s over which the data of this {@link SolrCollection} is distributed.
+   *
+   * <p>The map is from {@link Shard#getShardName()} to {@link Shard} instance.
+   */
+  Map<String, Shard> getShards();

Review comment:
       Please note that in some extreme (but real!) deployments the number of shards may be in the order of thousands and the number of replicas in the order of hundreds of thousands... Not so sure about a simple Map here - maybe a list of shard names + getShard accessor?
   
   (Or use a read-through pseudo-map like the one added with SOLR-14680 ?)

##########
File path: solr/core/src/java/org/apache/solr/cluster/placement/impl/PlacementPluginFactoryFacade.java
##########
@@ -0,0 +1,75 @@
+/*
+ * 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.solr.cluster.placement.impl;
+
+import org.apache.solr.cluster.placement.PlacementPlugin;
+import org.apache.solr.cluster.placement.PlacementPluginConfig;
+import org.apache.solr.cluster.placement.PlacementPluginFactory;
+import org.apache.solr.common.SolrException;
+import org.apache.solr.core.PluginInfo;
+import org.apache.solr.core.SolrResourceLoader;
+
+/**
+ * <p>The internal class instantiating the configured {@link PlacementPluginFactory} and creating a {@link PlacementPlugin}
+ * instance by passing the the factory the appropriate configuration created from the {@code <placementPluginFactory>}
+ * element in {@code solr.xml}.
+ *
+ * <p>A single instance of {@link PlacementPlugin} is used for all placement computations and therefore must be reentrant.
+ * When configuration changes, a new instance of {@link PlacementPlugin} will be created by calling again
+ * {@link PlacementPluginFactory#createPluginInstance(PlacementPluginConfig)}.
+ */
+public class PlacementPluginFactoryFacade {

Review comment:
       I'm not convinced we need this as a part of the API - instead we could leave the details of `PlacementPlugin` life-cycle management to the `PlacementPluginFactory` implementation.

##########
File path: solr/core/src/java/org/apache/solr/cluster/placement/PropertyKeyFactory.java
##########
@@ -0,0 +1,61 @@
+/*
+ * 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.solr.cluster.placement;
+
+/**
+ * Factory used by the plugin to create property keys to request property values from Solr.<p>
+ *
+ * Building of a {@link PropertyKey} requires specifying the target (context) from which the value of that key should be
+ * obtained. This is done by specifying the appropriate {@link PropertyValueSource}.<br>
+ * For clarity, when only a single type of target is acceptable, the corresponding subtype of {@link PropertyValueSource} is used instead
+ * (for example {@link Node}).
+ */
+public interface PropertyKeyFactory {
+  /**
+   * Returns a property key to request the number of cores on a {@link Node}.
+   */
+  PropertyKey createCoreCountKey(Node node);

Review comment:
       I think this part of the API is the most controversial, as it indeed leads to a large number of interfaces and impl. classes.
   
   On one hand this indeed provides the super-strong typing, the burden is on Solr-core to provide the implementation of these interfaces so it doesn't complicate the lives of plugin devs. But on the other hand this may be taking things too far ...
   
   I don't see yet how to simplify it without compromising the strong-typing too much - maybe just use a single `PropertyKey` interface + enum for key types and use one method like `PropertyKey createPropertyKey(PropertyKey.Type type, Node node, Object... params)`? This of course compromises the strong-typing, as it uses weakly-typed params and requires callers to know what parameters to supply ... so I'm not sure. But it would reduce the API surface significantly so maybe it's a compromise worth taking.




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@lucene.apache.org
For additional commands, e-mail: issues-help@lucene.apache.org