Skip to content

Commit

Permalink
SOLR-17096: Cluster Singleton plugin support in solr.xml (#2126)
Browse files Browse the repository at this point in the history
* Generalize to all "cluster plugins", albeit only supporting <clusterSingleton> and <replicaPlacementFactory> at this time.
* Introduced new `solr.cluster.plugin.edit.enabled` boolean setting to disable /cluster/plugin mutability, required for solr.xml's use of those plugins.

Co-authored-by: Paul McArthur <pmcarthur-apache@proton.me>
  • Loading branch information
pjmcarthur and pmcarthur-apache committed Jan 17, 2024
1 parent 6a84361 commit 1de6d08
Show file tree
Hide file tree
Showing 18 changed files with 870 additions and 160 deletions.
2 changes: 2 additions & 0 deletions solr/CHANGES.txt
Original file line number Diff line number Diff line change
Expand Up @@ -23,6 +23,8 @@ Improvements

* SOLR-17077: When a shard rejoins leader election, leave previous election only once to save unneeded calls to Zookeeper. (Pierre Salagnac)

* SOLR-17096: solr.xml now supports declaring clusterSingleton plugins (Paul McArthur, David Smiley)

Optimizations
---------------------
(No changes)
Expand Down
83 changes: 83 additions & 0 deletions solr/core/src/java/org/apache/solr/api/ClusterPluginsSource.java
Original file line number Diff line number Diff line change
@@ -0,0 +1,83 @@
/*
* 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.api;

import java.io.IOException;
import java.util.Map;
import java.util.function.Function;
import org.apache.solr.client.solrj.request.beans.PluginMeta;
import org.apache.solr.common.util.EnvUtils;
import org.apache.solr.core.CoreContainer;
import org.apache.solr.core.SolrResourceLoader;
import org.apache.solr.handler.admin.ContainerPluginsApi;

/** A source for Cluster Plugin configurations */
public interface ClusterPluginsSource {

/**
* Resolves the name of the class that will be used to provide cluster plugins.
*
* @return The name of the class to use as the {@link ClusterPluginsSource}
*/
public static String resolveClassName() {
return EnvUtils.getPropAsBool(ContainerPluginsRegistry.CLUSTER_PLUGIN_EDIT_ENABLED, true)
? ZkClusterPluginsSource.class.getName()
: NodeConfigClusterPluginsSource.class.getName();
}

public static ClusterPluginsSource loadClusterPluginsSource(
CoreContainer cc, SolrResourceLoader loader) {
return loader.newInstance(
resolveClassName(),
ClusterPluginsSource.class,
new String[0],
new Class<?>[] {CoreContainer.class},
new Object[] {cc});
}

/**
* Get the Read Api for this plugin source
*
* @return A {@link ContainerPluginsApi} Read Api for this plugin source
*/
ContainerPluginsApi.Read getReadApi();

/**
* Get the Edit Api for this plugin source, if it supports edit operations
*
* @return A {@link ContainerPluginsApi} Edit Api for this plugin source, or null if the plugin
* source does not support editing the plugin configs
*/
ContainerPluginsApi.Edit getEditApi();

/**
* Get a map of cluster plugin configurations from this source, where keys are plugin names and
* values are {@link PluginMeta} plugin metadata.
*
* @return An immutable map of plugin configurations
*/
Map<String, Object> plugins() throws IOException;

/**
* Persist the updated set of plugin configs
*
* @param modifier A function providing the map of plugin configs to be persisted
*/
void persistPlugins(Function<Map<String, Object>, Map<String, Object>> modifier)
throws IOException;
}
Original file line number Diff line number Diff line change
Expand Up @@ -39,7 +39,6 @@
import java.util.Optional;
import java.util.concurrent.CopyOnWriteArrayList;
import java.util.concurrent.Phaser;
import java.util.function.Supplier;
import org.apache.lucene.util.ResourceLoaderAware;
import org.apache.solr.client.solrj.SolrRequest;
import org.apache.solr.client.solrj.request.beans.PluginMeta;
Expand All @@ -55,7 +54,6 @@
import org.apache.solr.common.util.Utils;
import org.apache.solr.core.CoreContainer;
import org.apache.solr.core.PluginInfo;
import org.apache.solr.handler.admin.ContainerPluginsApi;
import org.apache.solr.pkg.SolrPackageLoader;
import org.apache.solr.request.SolrQueryRequest;
import org.apache.solr.response.SolrQueryResponse;
Expand All @@ -66,7 +64,7 @@
/**
* This class manages the container-level plugins and their Api-s. It is responsible for adding /
* removing / replacing the plugins according to the updated configuration obtained from {@link
* ContainerPluginsApi#plugins(Supplier)}.
* ClusterPluginsSource#plugins()}.
*
* <p>Plugins instantiated by this class may implement zero or more {@link Api}-s, which are then
* registered in the CoreContainer {@link ApiBag}. They may be also post-processed for additional
Expand All @@ -75,6 +73,8 @@
public class ContainerPluginsRegistry implements ClusterPropertiesListener, MapWriter, Closeable {
private static final Logger log = LoggerFactory.getLogger(MethodHandles.lookup().lookupClass());

public static final String CLUSTER_PLUGIN_EDIT_ENABLED = "solr.cluster.plugin.edit.enabled";

private static final ObjectMapper mapper =
SolrJacksonAnnotationInspector.createObjectMapper()
.configure(DeserializationFeature.FAIL_ON_UNKNOWN_PROPERTIES, false)
Expand All @@ -85,6 +85,8 @@ public class ContainerPluginsRegistry implements ClusterPropertiesListener, MapW
private final CoreContainer coreContainer;
private final ApiBag containerApiBag;

private final ClusterPluginsSource pluginsSource;

private final Map<String, ApiInfo> currentPlugins = new HashMap<>();

private Phaser phaser;
Expand Down Expand Up @@ -117,9 +119,11 @@ public void unregisterListener(PluginRegistryListener listener) {
listeners.remove(listener);
}

public ContainerPluginsRegistry(CoreContainer coreContainer, ApiBag apiBag) {
public ContainerPluginsRegistry(
CoreContainer coreContainer, ApiBag apiBag, ClusterPluginsSource pluginsSource) {
this.coreContainer = coreContainer;
this.containerApiBag = apiBag;
this.pluginsSource = pluginsSource;
}

@Override
Expand Down Expand Up @@ -171,7 +175,7 @@ public int hashCode() {
public synchronized void refresh() {
Map<String, Object> pluginInfos;
try {
pluginInfos = ContainerPluginsApi.plugins(coreContainer.zkClientSupplier);
pluginInfos = pluginsSource.plugins();
} catch (IOException e) {
log.error("Could not read plugins data", e);
return;
Expand Down
Original file line number Diff line number Diff line change
@@ -0,0 +1,106 @@
/*
* 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.api;

import java.io.IOException;
import java.util.HashMap;
import java.util.Map;
import java.util.function.Function;
import org.apache.solr.cluster.placement.PlacementPluginFactory;
import org.apache.solr.core.CoreContainer;
import org.apache.solr.core.NodeConfig;
import org.apache.solr.core.PluginInfo;
import org.apache.solr.handler.admin.ContainerPluginsApi;

/**
* Plugin configurations that are defined in solr.xml. This supports immutable deployments, and the
* /cluster/plugin Edit APIs are not available
*/
public class NodeConfigClusterPluginsSource implements ClusterPluginsSource {

private final Map<String, Object> plugins;

private final ContainerPluginsApi api;

public NodeConfigClusterPluginsSource(final CoreContainer cc) {
api = new ContainerPluginsApi(cc, this);
plugins = Map.copyOf(readPlugins(cc.getNodeConfig()));
}

@Override
public ContainerPluginsApi.Read getReadApi() {
return api.readAPI;
}

@Override
public ContainerPluginsApi.Edit getEditApi() {
return null;
}

@Override
public Map<String, Object> plugins() throws IOException {
return plugins;
}

/**
* This method should never be invoked because the Edit Apis are not made available by the plugin
*
* @throws UnsupportedOperationException always
*/
@Override
public void persistPlugins(Function<Map<String, Object>, Map<String, Object>> modifier) {
throw new UnsupportedOperationException(
"The NodeConfigContainerPluginsSource does not support updates to plugin configurations");
}

private static Map<String, Object> readPlugins(final NodeConfig cfg) {
Map<String, Object> pluginInfos = new HashMap<>();
PluginInfo[] clusterPlugins = cfg.getClusterPlugins();
if (clusterPlugins != null) {
for (PluginInfo p : clusterPlugins) {
Map<String, Object> pluginMap = new HashMap<>();
final String pluginName = getPluginName(p);
pluginMap.put("name", pluginName);
pluginMap.put("class", p.className);

if (p.initArgs.size() > 0) {
Map<String, Object> config = p.initArgs.toMap(new HashMap<>());
pluginMap.put("config", config);
}

pluginInfos.put(pluginName, pluginMap);
}
}
return pluginInfos;
}

/**
* Get the correct name for a plugin. Custom plugins must have a name set already, but built-in
* plugins may omit the name in solr.xml and require inference here
*/
private static String getPluginName(final PluginInfo pluginInfo) {

if (pluginInfo.name == null) {
if (pluginInfo.type.equals("replicaPlacementFactory")) {
return PlacementPluginFactory.PLUGIN_NAME;
}
}

return pluginInfo.name;
}
}
114 changes: 114 additions & 0 deletions solr/core/src/java/org/apache/solr/api/ZkClusterPluginsSource.java
Original file line number Diff line number Diff line change
@@ -0,0 +1,114 @@
/*
* 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.api;

import java.io.IOException;
import java.util.LinkedHashMap;
import java.util.Map;
import java.util.function.Function;
import java.util.function.Supplier;
import org.apache.solr.client.solrj.request.beans.PluginMeta;
import org.apache.solr.common.cloud.SolrZkClient;
import org.apache.solr.common.cloud.ZkStateReader;
import org.apache.solr.common.util.Utils;
import org.apache.solr.core.CoreContainer;
import org.apache.solr.handler.admin.ContainerPluginsApi;
import org.apache.zookeeper.KeeperException;
import org.apache.zookeeper.data.Stat;

/**
* The plugin configurations are stored and retrieved from the ZooKeeper cluster properties, stored
* at the {@link ZkStateReader#CONTAINER_PLUGINS} location This supports mutable configurations, and
* management via the /cluster/plugin APIs
*/
public class ZkClusterPluginsSource implements ClusterPluginsSource {

private final Supplier<SolrZkClient> zkClientSupplier;

private final ContainerPluginsApi api;

public ZkClusterPluginsSource(CoreContainer coreContainer) {
this.zkClientSupplier = coreContainer.zkClientSupplier;
this.api = new ContainerPluginsApi(coreContainer, this);
}

@Override
public ContainerPluginsApi.Read getReadApi() {
return api.readAPI;
}

@Override
public ContainerPluginsApi.Edit getEditApi() {
return api.editAPI;
}

/**
* Retrieve the current plugin configurations.
*
* @return current plugin configurations, where keys are plugin names and values are {@link
* PluginMeta} plugin metadata.
* @throws IOException on IO errors
*/
@Override
@SuppressWarnings("unchecked")
public Map<String, Object> plugins() throws IOException {
SolrZkClient zkClient = zkClientSupplier.get();
try {
Map<String, Object> clusterPropsJson =
(Map<String, Object>)
Utils.fromJSON(zkClient.getData(ZkStateReader.CLUSTER_PROPS, null, new Stat(), true));
return Map.copyOf(
(Map<String, Object>)
clusterPropsJson.computeIfAbsent(
ZkStateReader.CONTAINER_PLUGINS, o -> new LinkedHashMap<>()));
} catch (KeeperException.NoNodeException e) {
return new LinkedHashMap<>();
} catch (KeeperException | InterruptedException e) {
throw new IOException("Error reading cluster property", SolrZkClient.checkInterrupted(e));
}
}

@Override
public void persistPlugins(Function<Map<String, Object>, Map<String, Object>> modifier)
throws IOException {
try {
zkClientSupplier
.get()
.atomicUpdate(
ZkStateReader.CLUSTER_PROPS,
bytes -> {
@SuppressWarnings("unchecked")
Map<String, Object> rawJson =
bytes == null
? new LinkedHashMap<>()
: (Map<String, Object>) Utils.fromJSON(bytes);
@SuppressWarnings("unchecked")
Map<String, Object> pluginsModified =
modifier.apply(
(Map<String, Object>)
rawJson.computeIfAbsent(
ZkStateReader.CONTAINER_PLUGINS, o -> new LinkedHashMap<>()));
if (pluginsModified == null) return null;
rawJson.put(ZkStateReader.CONTAINER_PLUGINS, pluginsModified);
return Utils.toJSON(rawJson);
});
} catch (KeeperException | InterruptedException e) {
throw new IOException("Error reading cluster property", SolrZkClient.checkInterrupted(e));
}
}
}

0 comments on commit 1de6d08

Please sign in to comment.