You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@lucene.apache.org by ab...@apache.org on 2021/02/01 15:50:40 UTC

[lucene-solr] branch master updated: SOLR-15068: RefGuide documentation for replica placement plugins (plus minor cleanups).

This is an automated email from the ASF dual-hosted git repository.

ab pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/lucene-solr.git


The following commit(s) were added to refs/heads/master by this push:
     new 9e8ca98  SOLR-15068: RefGuide documentation for replica placement plugins (plus minor cleanups).
9e8ca98 is described below

commit 9e8ca98985fb4615475bc43b85a551afe5800d8e
Author: Andrzej Bialecki <ab...@apache.org>
AuthorDate: Mon Feb 1 16:49:01 2021 +0100

    SOLR-15068: RefGuide documentation for replica placement plugins (plus
    minor cleanups).
---
 .../placement/plugins/AffinityPlacementConfig.java |  18 +-
 .../plugins/AffinityPlacementFactory.java          |   4 +-
 solr/solr-ref-guide/src/cluster-plugins.adoc       |  10 +-
 .../src/replica-placement-plugins.adoc             | 239 +++++++++++++++++++++
 solr/solr-ref-guide/src/solr-plugins.adoc          |   5 +-
 5 files changed, 264 insertions(+), 12 deletions(-)

diff --git a/solr/core/src/java/org/apache/solr/cluster/placement/plugins/AffinityPlacementConfig.java b/solr/core/src/java/org/apache/solr/cluster/placement/plugins/AffinityPlacementConfig.java
index b45e6a9..d9579bc 100644
--- a/solr/core/src/java/org/apache/solr/cluster/placement/plugins/AffinityPlacementConfig.java
+++ b/solr/core/src/java/org/apache/solr/cluster/placement/plugins/AffinityPlacementConfig.java
@@ -28,7 +28,11 @@ import java.util.Objects;
  */
 public class AffinityPlacementConfig implements PlacementPluginConfig {
 
-  public static final AffinityPlacementConfig DEFAULT = new AffinityPlacementConfig(20L, 100L);
+  public static final long DEFAULT_MINIMAL_FREE_DISK_GB = 20L;
+  public static final long DEFAULT_PRIORITIZED_FREE_DISK_GB = 100L;
+
+  public static final AffinityPlacementConfig DEFAULT =
+      new AffinityPlacementConfig(DEFAULT_MINIMAL_FREE_DISK_GB, DEFAULT_PRIORITIZED_FREE_DISK_GB);
 
   /**
    * If a node has strictly less GB of free disk than this value, the node is excluded from assignment decisions.
@@ -53,13 +57,13 @@ public class AffinityPlacementConfig implements PlacementPluginConfig {
    * they are not already present.
    */
   @JsonProperty
-  public Map<String, String> withCollections;
+  public Map<String, String> withCollection;
 
   /**
    * Zero-arguments public constructor required for deserialization - don't use.
    */
   public AffinityPlacementConfig() {
-    this(0L, 0L);
+    this(DEFAULT_MINIMAL_FREE_DISK_GB, DEFAULT_PRIORITIZED_FREE_DISK_GB);
   }
 
   /**
@@ -75,14 +79,14 @@ public class AffinityPlacementConfig implements PlacementPluginConfig {
    * Configuration for the {@link AffinityPlacementFactory}.
    * @param minimalFreeDiskGB minimal free disk GB.
    * @param prioritizedFreeDiskGB prioritized free disk GB.
-   * @param withCollections configuration of co-located collections: keys are
+   * @param withCollection configuration of co-located collections: keys are
    *                        primary collection names and values are secondary
    *                        collection names.
    */
-  public AffinityPlacementConfig(long minimalFreeDiskGB, long prioritizedFreeDiskGB, Map<String, String> withCollections) {
+  public AffinityPlacementConfig(long minimalFreeDiskGB, long prioritizedFreeDiskGB, Map<String, String> withCollection) {
     this.minimalFreeDiskGB = minimalFreeDiskGB;
     this.prioritizedFreeDiskGB = prioritizedFreeDiskGB;
-    Objects.requireNonNull(withCollections);
-    this.withCollections = withCollections;
+    Objects.requireNonNull(withCollection);
+    this.withCollection = withCollection;
   }
 }
diff --git a/solr/core/src/java/org/apache/solr/cluster/placement/plugins/AffinityPlacementFactory.java b/solr/core/src/java/org/apache/solr/cluster/placement/plugins/AffinityPlacementFactory.java
index 79d1f92..eaec4ab 100644
--- a/solr/core/src/java/org/apache/solr/cluster/placement/plugins/AffinityPlacementFactory.java
+++ b/solr/core/src/java/org/apache/solr/cluster/placement/plugins/AffinityPlacementFactory.java
@@ -108,7 +108,7 @@ import java.util.stream.Collectors;
  * </ul>
  *
  * <p>This code is a realistic placement computation, based on a few assumptions. The code is written in such a way to
- * make it relatively easy to adapt it to (somewhat) different assumptions. Configuration options could be introduced
+ * make it relatively easy to adapt it to (somewhat) different assumptions. Additional configuration options could be introduced
  * to allow configuration base option selection as well...</p>
  */
 public class AffinityPlacementFactory implements PlacementPluginFactory<AffinityPlacementConfig> {
@@ -149,7 +149,7 @@ public class AffinityPlacementFactory implements PlacementPluginFactory<Affinity
 
   @Override
   public PlacementPlugin createPluginInstance() {
-    return new AffinityPlacementPlugin(config.minimalFreeDiskGB, config.prioritizedFreeDiskGB, config.withCollections);
+    return new AffinityPlacementPlugin(config.minimalFreeDiskGB, config.prioritizedFreeDiskGB, config.withCollection);
   }
 
   @Override
diff --git a/solr/solr-ref-guide/src/cluster-plugins.adoc b/solr/solr-ref-guide/src/cluster-plugins.adoc
index 4ac7fce..8649ad6 100644
--- a/solr/solr-ref-guide/src/cluster-plugins.adoc
+++ b/solr/solr-ref-guide/src/cluster-plugins.adoc
@@ -69,7 +69,15 @@ Example plugin configuration:
 curl -X POST -H 'Content-type: application/json' -d '{
     "add":{
         "name": ".placement-plugin",
-        "class": "org.apache.solr.cluster.placement.impl.CollectionsRepairEventListener"
+        "class": "org.apache.solr.cluster.placement.plugins.AffinityPlacementFactory",
+        "config": {
+          "minimalFreeDiskGB": 20,
+          "prioritizedFreeDiskGB": 100,
+          "withCollection": {
+            "A_primary": "A_secondary",
+            "B_primary": "B_secondary"
+          }
+        }
     }}'
   http://localhost:8983/api/cluster/plugin
 ----
diff --git a/solr/solr-ref-guide/src/replica-placement-plugins.adoc b/solr/solr-ref-guide/src/replica-placement-plugins.adoc
new file mode 100644
index 0000000..7850d4c
--- /dev/null
+++ b/solr/solr-ref-guide/src/replica-placement-plugins.adoc
@@ -0,0 +1,239 @@
+= Replica Placement Plugins
+:toc: macro
+:toclevels: 5
+// 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.
+
+== Introduction
+When user wants to create a new collection or add a replica to one of the existing
+collections Solr needs to first determine on what node to put the replica so that the
+cluster resources are allocated in a well-balanced manner (according to some criteria).
+
+Replica placement plugin is the configurable component that determines these placements.
+It can also enforce additional constraints on operations such as collection or replica removal
+- for example if the plugin wants some collections to be always co-located on the same nodes,
+or always present when some other collection is present.
+
+(In Solr 8.x this functionality was using either per-collection rules, or it could be configured
+in the autoscaling framework).
+
+=== Plugin Configuration
+Replica placement plugin configurations are maintained using the `/cluster/plugin` API.
+There can be only one cluster-wide plugin configuration at a time, and it uses a pre-defined
+plugin name: `.placement-plugin`.
+
+There are several placement plugins included in the Solr distribution. Additional placement
+plugins can be added - they have to implement a `PlacementPluginFactory` interface. The
+configuration entry may also contain a `config` element if the plugin implements the
+`ConfigurablePlugin` interface.
+
+=== Legacy replica placement
+By default Solr 9.0 uses a legacy method for replica placements, which doesn't use a placement
+plugin at all. This method is used whenever the placement plugin configuration is missing or
+invalid.
+
+Legacy placement simply assigns new replicas to live nodes in a round-robin fashion: first it
+prepares a sorted list of nodes with the smallest number of existing replicas of the collection.
+Then for each shard in the request it adds the replicas to consecutive nodes in this order,
+wrapping around to the first node if the number of replicas is larger than the number of nodes.
+
+This placement strategy doesn't ensure that no more than 1 replica of a shard is placed on the
+same node. Also, the round-robin assignment only roughly approximates an even spread of replicas
+across the nodes.
+
+=== Placement plugins included in Solr
+The following placement plugins are available out-of-the-box in Solr 9.0 (however, as
+described above the default configuration doesn't use any of them, instead it uses the legacy
+replica placement method).
+
+In order to use a plugin its configuration must be added using the `/cluster/plugin` API.
+For example, in order to use the `AffinityPlacementFactory` plugin the following command
+should be executed:
+
+[source,bash]
+----
+curl -X POST -H 'Content-type: application/json' -d '{
+    "add":{
+        "name": ".placement-plugin",
+        "class": "org.apache.solr.cluster.placement.plugins.AffinityPlacementFactory",
+        "config": {
+          "minimalFreeDiskGB": 20,
+          "prioritizedFreeDiskGB": 100,
+          "withCollections": {
+            "A_primary": "A_secondary",
+            "B_primary": "B_secondary"
+          }
+        }
+    }}'
+  http://localhost:8983/api/cluster/plugin
+----
+
+Similarly, the configuration can be updated or removed.
+
+NOTE: Placement plugin configuration MUST use the predefined name `.placement-plugin`.
+There can be only one (or none) placement configuration defined.
+
+
+==== `RandomPlacementFactory`
+This plugin creates random placements, while preventing two replicas of the same shard from being
+placed on the same node. If there are too few nodes to satisfy these constraints an exception is
+thrown, and the request is rejected.
+
+Due to its simplistic algorithm this plugin should only be used in simple deployments.
+
+This plugin doesn't require any configuration.
+
+==== `MinimizeCoresPlacementFactory`
+This plugin creates placements that minimize the number of cores per node across all live nodes,
+while not placing two replicas of the same shard on the same node. If there are too few nodes
+to satisfy these constraints an exception is thrown, and the request is rejected.
+
+Due to its simplistic algorithm this plugin should only be used in simple deployments.
+
+This plugin doesn't require any configuration.
+
+==== `AffinityPlacementFactory`
+This plugin implements replica placement algorithm that roughly replicates Solr 8.x autoscaling
+configuration defined https://github.com/lucidworks/fusion-cloud-native/blob/master/policy.json#L16[here].
+
+The autoscaling specification in the configuration linked above aimed to do the following:
+
+* spread replicas per shard as evenly as possible across multiple availability zones (given by a system property),
+* assign replicas based on replica type to specific kinds of nodes (another system property), and
+* avoid having more than one replica per shard on the same node.
+* only after the above constraints are satisfied:
+** minimize cores per node, or
+** minimize disk usage.
+
+Additionally, it supports the `withCollection` constraint that enforces the placement of
+co-located collections' replicas on the same nodes, and prevents deletions of collections and
+replicas that would break this constraint.
+
+Overall strategy of this plugin:
+
+* The set of nodes in the cluster is obtained, and if the `withCollection` mapping is present
+  and applicable to the current collection then this candidate set is filtered so that only
+  eligible nodes remain according to this constraint.
+* The resulting node set is transformed into 3 independent sets (that can overlap) of nodes accepting each of the three replica types (NRT, TLOG and PULL).
+* For each shard on which placing replicas is required and then for each replica type to place (starting with NRT, then TLOG then PULL):
+** The set of candidates nodes corresponding to the replica type is used and from that set are removed nodes that already have a replica (of any type) for that shard
+** If there are not enough nodes, an error is thrown (this is checked further down during processing).
+** The number of (already existing) replicas of the current type on each Availability Zone is collected.
+** Separate the set of available nodes to as many subsets (possibly some are empty) as there are Availability Zones defined for the candidate nodes
+** In each AZ nodes subset, sort the nodes by increasing total number of cores count.
+** Iterate over the number of replicas to place (for the current replica type for the current shard):
+*** Based on the number of replicas per AZ collected previously, pick the non-empty set of nodes having the lowest number of replicas. Then pick the first node in that set. That's the node the replica is placed one. Remove the node from the set of available nodes for the given AZ and increase the number of replicas placed on that AZ.
+** During this process, the number of cores on the nodes in general is tracked to take into account placement decisions so that not all shards decide to put their replicas on the same nodes (they might though if these are the less loaded nodes).
+
+NOTE: At the moment the names of availability zone property and the name of the replica type
+property are not configurable, and set respectively to `availability_zone` and `replica_type`.
+
+===== `withCollection` constraint
+This plugin supports enforcing additional constraint named `withCollection`, which causes
+replicas of two paired collections to be placed on the same nodes.
+
+Users can define the collection pairs in the plugin configuration, in the `config/withCollection`
+element, which is a JSON map where keys are the primary collection names, and the values are the
+secondary collection names (currently only 1:1 mapping is supported - however, multiple primary
+collections may use the same secondary collection, which effectively relaxes this to N:1 mapping).
+
+Unlike previous versions of Solr, this plugin does NOT automatically place replicas of the
+secondary collection - those replicas are assumed to be already in place, and it's the
+responsibility of the user to already place them on the right nodes (most likely simply by
+using this plugin to create the secondary collection first, with large enough replication
+factor to ensure that the target node set is populated with secondary replicas).
+
+When a request to compute placements is processed for the primary collection that has a
+key in the `withCollection` map, the set of candidate nodes is first filtered to eliminate nodes
+that don't contain the replicas of the secondary collection. Please note that this may
+result in an empty set, and an exception - in this case the sufficient number of secondary
+replicas needs to be created first.
+
+The plugin preserves this co-location by rejecting delete operation of secondary collections
+(or their replicas) if they are still in use on the nodes where primary replicas are located
+- requests to do so will be rejected with errors. In order to delete a secondary collection
+(or its replicas) from these nodes first the replicas of the primary collection must be
+removed from the co-located nodes, or the configuration must be changed to remove the
+co-location mapping for the primary collection.
+
+===== Configuration
+This plugin supports the following configuration parameters:
+
+`minimalFreeDiskGB`::
+(optional, integer) if a node has strictly less GB of free disk than this value, the node is
+excluded from assignment decisions. Set to 0 or less to disable. Default value is 10.
+
+`prioritizedFreeDiskGB`::
+(optional, integer) replica allocation will assign replicas to nodes with at least this number
+of GB of free disk space regardless of the number of cores on these nodes rather than assigning
+replicas to nodes with less than this amount of free disk space if that's an option (if that's
+not an option, replicas can still be assigned to nodes with less than this amount of free space).
+Default value is 100.
+
+`withCollection`::
+(optional, map) this property defines additional constraints that primary collections (keys)
+must be located on the same nodes as the secondary collections (values). The plugin will
+assume that the secondary collection replicas are already in place and ignore candidate
+nodes where they are not already present. Default value is none.
+
+=== Example configurations
+This is a simple configuration that uses default values:
+
+[source,bash]
+----
+curl -X POST -H 'Content-type: application/json' -d '{
+    "add":{
+        "name": ".placement-plugin",
+        "class": "org.apache.solr.cluster.placement.plugins.AffinityPlacementFactory"
+    }}'
+  http://localhost:8983/api/cluster/plugin
+----
+
+This configuration specifies the base parameters:
+[source,bash]
+----
+curl -X POST -H 'Content-type: application/json' -d '{
+    "add":{
+        "name": ".placement-plugin",
+        "class": "org.apache.solr.cluster.placement.plugins.AffinityPlacementFactory",
+        "config": {
+          "minimalFreeDiskGB": 20,
+          "prioritizedFreeDiskGB": 100
+        }
+    }}'
+  http://localhost:8983/api/cluster/plugin
+----
+
+This configuration defines that collection `A_primary` must be co-located with
+collection `Common_secondary`, and collection `B_primary` must be co-located also with the
+collection `Common_secondary`:
+
+[source,bash]
+----
+curl -X POST -H 'Content-type: application/json' -d '{
+    "add":{
+        "name": ".placement-plugin",
+        "class": "org.apache.solr.cluster.placement.plugins.AffinityPlacementFactory",
+        "config": {
+          "withCollection": {
+            "A_primary": "Common_secondary",
+            "B_primary": "Common_secondary"
+          }
+        }
+    }}'
+  http://localhost:8983/api/cluster/plugin
+----
\ No newline at end of file
diff --git a/solr/solr-ref-guide/src/solr-plugins.adoc b/solr/solr-ref-guide/src/solr-plugins.adoc
index d9946b2..3d12f50 100644
--- a/solr/solr-ref-guide/src/solr-plugins.adoc
+++ b/solr/solr-ref-guide/src/solr-plugins.adoc
@@ -1,7 +1,8 @@
 = Solr Plugins
 :page-children: libs, \
     package-manager, \
-    cluster-plugins
+    cluster-plugins, \
+    replica-placement-plugins
 // 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
@@ -59,4 +60,4 @@ Only some plugins support this as of now (support for more types of plugins comi
 
 * <<cluster-plugins.adoc#cluster-plugins,Cluster Plugins>>:
 Describes the API used for managing cluster-level plugins such as request handlers,
-cluster-level event producer and replica placement plugins.
+cluster-level event producer and <<replica-placement-plugins.adoc#replica-placement-plugins,replica placement plugins>>.