diff --git a/.run/NBCLI web foreground dryrun.run.xml b/.run/NBCLI web foreground dryrun.run.xml deleted file mode 100644 index 4180a9a89..000000000 --- a/.run/NBCLI web foreground dryrun.run.xml +++ /dev/null @@ -1,16 +0,0 @@ - - - - \ No newline at end of file diff --git a/driver-jms/pom.xml b/adapter-pulsar/pom.xml similarity index 58% rename from driver-jms/pom.xml rename to adapter-pulsar/pom.xml index c3cbcc803..2428c54fe 100644 --- a/driver-jms/pom.xml +++ b/adapter-pulsar/pom.xml @@ -17,61 +17,49 @@ 4.0.0 + adapter-pulsar + jar + mvn-defaults io.nosqlbench - 4.17.22-SNAPSHOT + 4.17.31-SNAPSHOT ../mvn-defaults - driver-jms - jar ${project.artifactId} - - A JMS driver for nosqlbench. This provides the ability to inject synthetic data - into a pulsar system via JMS 2.0 compatibile APIs. - - NOTE: this is JMS compatible driver from DataStax that allows using a Pulsar cluster - as the potential JMS Destination + A Pulsar driver for nosqlbench. This provides the ability to inject synthetic data + into a pulsar system. - - - - - - - - - - - - - - + + 2.10.1 + - io.nosqlbench engine-api - 4.17.22-SNAPSHOT + 4.17.31-SNAPSHOT - - org.apache.commons - commons-lang3 - 3.12.0 + io.nosqlbench + adapters-api + 4.17.31-SNAPSHOT - - org.projectlombok - lombok - 1.18.24 - provided + org.apache.pulsar + pulsar-client + ${pulsar.version} + + + + org.apache.pulsar + pulsar-client-admin + ${pulsar.version} @@ -88,13 +76,19 @@ 2.8.0 - + - com.datastax.oss - pulsar-jms - 2.4.11 + org.apache.avro + avro + 1.11.1 + + + org.apache.commons + commons-lang3 + 3.12.0 + diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/PulsarDriverAdapter.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/PulsarDriverAdapter.java new file mode 100644 index 000000000..4d0c2c7fb --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/PulsarDriverAdapter.java @@ -0,0 +1,54 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar; + +import io.nosqlbench.adapter.pulsar.ops.PulsarOp; +import io.nosqlbench.engine.api.activityimpl.OpMapper; +import io.nosqlbench.engine.api.activityimpl.uniform.BaseDriverAdapter; +import io.nosqlbench.engine.api.activityimpl.uniform.DriverAdapter; +import io.nosqlbench.engine.api.activityimpl.uniform.DriverSpaceCache; +import io.nosqlbench.nb.annotations.Maturity; +import io.nosqlbench.nb.annotations.Service; +import io.nosqlbench.api.config.standard.NBConfigModel; +import io.nosqlbench.api.config.standard.NBConfiguration; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; + +import java.util.function.Function; + +@Service(value = DriverAdapter.class, selector = "pulsar") +public class PulsarDriverAdapter extends BaseDriverAdapter { + + private final static Logger logger = LogManager.getLogger(PulsarDriverAdapter.class); + + @Override + public OpMapper getOpMapper() { + DriverSpaceCache spaceCache = getSpaceCache(); + NBConfiguration adapterConfig = getConfiguration(); + return new PulsarOpMapper(this, adapterConfig, spaceCache); + } + + @Override + public Function getSpaceInitializer(NBConfiguration cfg) { + return (s) -> new PulsarSpace(s, cfg); + } + + @Override + public NBConfigModel getConfigModel() { + return super.getConfigModel().add(PulsarSpace.getConfigModel()); + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/PulsarOpMapper.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/PulsarOpMapper.java new file mode 100644 index 000000000..a3db586ee --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/PulsarOpMapper.java @@ -0,0 +1,82 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar; + +import io.nosqlbench.adapter.pulsar.dispensers.*; +import io.nosqlbench.adapter.pulsar.ops.PulsarOp; +import io.nosqlbench.api.config.standard.NBConfiguration; +import io.nosqlbench.engine.api.activityimpl.OpDispenser; +import io.nosqlbench.engine.api.activityimpl.OpMapper; +import io.nosqlbench.engine.api.activityimpl.uniform.DriverAdapter; +import io.nosqlbench.engine.api.activityimpl.uniform.DriverSpaceCache; +import io.nosqlbench.engine.api.templating.ParsedOp; +import io.nosqlbench.engine.api.templating.TypeAndTarget; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; + +public class PulsarOpMapper implements OpMapper { + + private final static Logger logger = LogManager.getLogger(PulsarOpMapper.class); + + private final NBConfiguration cfg; + private final DriverSpaceCache spaceCache; + private final DriverAdapter adapter; + + public PulsarOpMapper(DriverAdapter adapter, NBConfiguration cfg, DriverSpaceCache spaceCache) { + this.cfg = cfg; + this.spaceCache = spaceCache; + this.adapter = adapter; + } + + @Override + public OpDispenser apply(ParsedOp op) { + String spaceName = op.getStaticConfigOr("space", "default"); + PulsarSpace pulsarSpace = spaceCache.get(spaceName); + + /* + * If the user provides a body element, then they want to provide the JSON or + * a data structure that can be converted into JSON, bypassing any further + * specialized type-checking or op-type specific features + */ + if (op.isDefined("body")) { + throw new RuntimeException("This mode is reserved for later. Do not use the 'body' op field."); + } + else { + TypeAndTarget opType = op.getTypeAndTarget(PulsarOpType.class, String.class); + + return switch (opType.enumId) { + case AdminTenant -> + new AdminTenantOpDispenser(adapter, op, opType.targetFunction, pulsarSpace); + case AdminNamespace -> + new AdminNamespaceOpDispenser(adapter, op, opType.targetFunction, pulsarSpace); + case AdminTopic -> + new AdminTopicOpDispenser(adapter, op, opType.targetFunction, pulsarSpace); + case MessageProduce -> + new MessageProducerOpDispenser(adapter, op, opType.targetFunction, pulsarSpace); + case MessageConsume -> + new MessageConsumerOpDispenser(adapter, op, opType.targetFunction, pulsarSpace); + ////////////////////////// + // NOTE: not sure how useful to have Pulsar message reader API in the NB performance testing + // currently, the reader API in NB Pulsar driver is no-op (see TDOD in MessageReaderOp) + ////////////////////////// + case MessageRead -> + new MessageReaderOpDispenser(adapter, op, opType.targetFunction, pulsarSpace); + }; + } + } + +} diff --git a/driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsTimeTrackOp.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/PulsarOpType.java similarity index 62% rename from driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsTimeTrackOp.java rename to adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/PulsarOpType.java index 8d588faff..d63185121 100644 --- a/driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsTimeTrackOp.java +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/PulsarOpType.java @@ -14,20 +14,16 @@ * limitations under the License. */ -package io.nosqlbench.driver.jms.ops; +package io.nosqlbench.adapter.pulsar; -/** - * Base type of all Sync Pulsar Operations including Producers and Consumers. - */ -public abstract class JmsTimeTrackOp implements JmsOp { - - public void run(Runnable timeTracker) { - try { - this.run(); - } finally { - timeTracker.run(); - } - } - - public abstract void run(); +public enum PulsarOpType { + AdminTenant, + AdminNamespace, + AdminTopic, + MessageProduce, + // This also supports multi-topic message consumption + MessageConsume, + MessageRead; } + + diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/PulsarSpace.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/PulsarSpace.java new file mode 100644 index 000000000..708e26a96 --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/PulsarSpace.java @@ -0,0 +1,228 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar; + +import io.nosqlbench.adapter.pulsar.exception.PulsarAdapterUnexpectedException; +import io.nosqlbench.adapter.pulsar.util.PulsarAdapterUtil; +import io.nosqlbench.adapter.pulsar.util.PulsarClientConf; +import io.nosqlbench.api.config.standard.ConfigModel; +import io.nosqlbench.api.config.standard.NBConfigModel; +import io.nosqlbench.api.config.standard.NBConfiguration; +import io.nosqlbench.api.config.standard.Param; +import org.apache.commons.lang3.BooleanUtils; +import org.apache.commons.lang3.StringUtils; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; +import org.apache.pulsar.client.admin.PulsarAdmin; +import org.apache.pulsar.client.admin.PulsarAdminBuilder; +import org.apache.pulsar.client.api.*; +import org.apache.pulsar.common.schema.KeyValueEncodingType; + +import java.util.Map; +import java.util.concurrent.ConcurrentHashMap; + +public class PulsarSpace implements AutoCloseable { + + private final static Logger logger = LogManager.getLogger(PulsarSpace.class); + + private final String spaceName; + private final NBConfiguration cfg; + + private final String pulsarSvcUrl; + private final String webSvcUrl; + + private PulsarClientConf pulsarClientConf; + private PulsarClient pulsarClient; + private PulsarAdmin pulsarAdmin; + private Schema pulsarSchema; + + private final ConcurrentHashMap> producers = new ConcurrentHashMap<>(); + private final ConcurrentHashMap> consumers = new ConcurrentHashMap<>(); + private final ConcurrentHashMap> readers = new ConcurrentHashMap<>(); + + + public PulsarSpace(String spaceName, NBConfiguration cfg) { + this.spaceName = spaceName; + this.cfg = cfg; + + this.pulsarSvcUrl = cfg.get("service_url"); + this.webSvcUrl = cfg.get("web_url"); + this.pulsarClientConf = new PulsarClientConf(cfg.get("config")); + + initPulsarAdminAndClientObj(); + createPulsarSchemaFromConf(); + } + + public static NBConfigModel getConfigModel() { + return ConfigModel.of(PulsarSpace.class) + .add(Param.defaultTo("service_url", "pulsar://localhost:6650") + .setDescription("Pulsar broker service URL.")) + .add(Param.defaultTo("web_url", "http://localhost:8080") + .setDescription("Pulsar web service URL.")) + .add(Param.defaultTo("config", "config.properties") + .setDescription("Pulsar client connection configuration property file.")) + .add(Param.defaultTo("cyclerate_per_thread", false) + .setDescription("Apply cycle rate per NB thread")) + .asReadOnly(); + } + + public String getPulsarSvcUrl() { return pulsarSvcUrl; } + public String getWebSvcUrl() { return webSvcUrl; } + public PulsarClientConf getPulsarNBClientConf() { return pulsarClientConf; } + public PulsarClient getPulsarClient() { return pulsarClient; } + public PulsarAdmin getPulsarAdmin() { return pulsarAdmin; } + public Schema getPulsarSchema() { return pulsarSchema; } + public int getProducerSetCnt() { return producers.size(); } + public int getConsumerSetCnt() { return consumers.size(); } + public int getReaderSetCnt() { return readers.size(); } + public Producer getProducer(String name) { return producers.get(name); } + public void setProducer(String name, Producer producer) { producers.put(name, producer); } + public Consumer getConsumer(String name) { return consumers.get(name); } + public void setConsumer(String name, Consumer consumer) { consumers.put(name, consumer); } + + public Reader getReader(String name) { return readers.get(name); } + public void setReader(String name, Reader reader) { readers.put(name, reader); } + + + /** + * Initialize + * - PulsarAdmin object for adding/deleting tenant, namespace, and topic + * - PulsarClient object for message publishing and consuming + */ + private void initPulsarAdminAndClientObj() { + PulsarAdminBuilder adminBuilder = + PulsarAdmin.builder() + .serviceHttpUrl(webSvcUrl); + + ClientBuilder clientBuilder = PulsarClient.builder(); + + try { + Map clientConfMap = pulsarClientConf.getClientConfMapRaw(); + + // Override "client.serviceUrl" setting in config.properties + clientConfMap.remove("serviceUrl"); + clientBuilder.loadConf(clientConfMap).serviceUrl(pulsarSvcUrl); + + // Pulsar Authentication + String authPluginClassName = + pulsarClientConf.getClientConfValue(PulsarAdapterUtil.CLNT_CONF_KEY.authPulginClassName.label); + String authParams = + pulsarClientConf.getClientConfValue(PulsarAdapterUtil.CLNT_CONF_KEY.authParams.label); + + if ( !StringUtils.isAnyBlank(authPluginClassName, authParams) ) { + adminBuilder.authentication(authPluginClassName, authParams); + clientBuilder.authentication(authPluginClassName, authParams); + } + + boolean useTls = StringUtils.contains(pulsarSvcUrl, "pulsar+ssl"); + if ( useTls ) { + String tlsHostnameVerificationEnableStr = + pulsarClientConf.getClientConfValue(PulsarAdapterUtil.CLNT_CONF_KEY.tlsHostnameVerificationEnable.label); + boolean tlsHostnameVerificationEnable = BooleanUtils.toBoolean(tlsHostnameVerificationEnableStr); + + adminBuilder + .enableTlsHostnameVerification(tlsHostnameVerificationEnable); + clientBuilder + .enableTlsHostnameVerification(tlsHostnameVerificationEnable); + + String tlsTrustCertsFilePath = + pulsarClientConf.getClientConfValue(PulsarAdapterUtil.CLNT_CONF_KEY.tlsTrustCertsFilePath.label); + if (!StringUtils.isBlank(tlsTrustCertsFilePath)) { + adminBuilder.tlsTrustCertsFilePath(tlsTrustCertsFilePath); + clientBuilder.tlsTrustCertsFilePath(tlsTrustCertsFilePath); + } + + String tlsAllowInsecureConnectionStr = + pulsarClientConf.getClientConfValue(PulsarAdapterUtil.CLNT_CONF_KEY.tlsAllowInsecureConnection.label); + boolean tlsAllowInsecureConnection = BooleanUtils.toBoolean(tlsAllowInsecureConnectionStr); + adminBuilder.allowTlsInsecureConnection(tlsAllowInsecureConnection); + clientBuilder.allowTlsInsecureConnection(tlsAllowInsecureConnection); + } + + pulsarAdmin = adminBuilder.build(); + pulsarClient = clientBuilder.build(); + + } catch (PulsarClientException e) { + logger.error("Fail to create PulsarAdmin and/or PulsarClient object from the global configuration!"); + throw new RuntimeException("Fail to create PulsarAdmin and/or PulsarClient object from global configuration!"); + } + } + + public void shutdownSpace() { + try { + for (Producer producer : producers.values()) { + if (producer != null) producer.close(); + } + for (Consumer consumer : consumers.values()) { + if (consumer != null) consumer.close(); + } + for (Reader reader : readers.values()) { + if (reader != null) reader.close(); + } + if (pulsarAdmin != null) pulsarAdmin.close(); + if (pulsarClient != null) pulsarClient.close(); + } + catch (Exception e) { + throw new PulsarAdapterUnexpectedException( + "Unexpected error when shutting down the Pulsar space \"" + spaceName + "\"!"); + } + } + + /** + * Get Pulsar schema from the definition string + */ + + private Schema buildSchemaFromDefinition(String schemaTypeConfEntry, + String schemaDefinitionConfEntry) { + String schemaType = pulsarClientConf.getSchemaConfValue(schemaTypeConfEntry); + String schemaDef = pulsarClientConf.getSchemaConfValue(schemaDefinitionConfEntry); + + Schema result; + if (PulsarAdapterUtil.isAvroSchemaTypeStr(schemaType)) { + result = PulsarAdapterUtil.getAvroSchema(schemaType, schemaDef); + } else if (PulsarAdapterUtil.isPrimitiveSchemaTypeStr(schemaType)) { + result = PulsarAdapterUtil.getPrimitiveTypeSchema(schemaType); + } else if (PulsarAdapterUtil.isAutoConsumeSchemaTypeStr(schemaType)) { + result = Schema.AUTO_CONSUME(); + } else { + throw new RuntimeException("Unsupported schema type string: " + schemaType + "; " + + "Only primitive type, Avro type and AUTO_CONSUME are supported at the moment!"); + } + return result; + } + private void createPulsarSchemaFromConf() { + pulsarSchema = buildSchemaFromDefinition("schema.type", "schema.definition"); + + // this is to allow KEY_VALUE schema + if (pulsarClientConf.hasSchemaConfKey("schema.key.type")) { + Schema pulsarKeySchema = buildSchemaFromDefinition("schema.key.type", "schema.key.definition"); + String encodingType = pulsarClientConf.getSchemaConfValue("schema.keyvalue.encodingtype"); + KeyValueEncodingType keyValueEncodingType = KeyValueEncodingType.SEPARATED; + if (encodingType != null) { + keyValueEncodingType = KeyValueEncodingType.valueOf(encodingType); + } + pulsarSchema = Schema.KeyValue(pulsarKeySchema, pulsarSchema, keyValueEncodingType); + } + } + + @Override + public void close() { + shutdownSpace(); + } +} + + diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/AdminNamespaceOpDispenser.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/AdminNamespaceOpDispenser.java new file mode 100644 index 000000000..eb41ce1ed --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/AdminNamespaceOpDispenser.java @@ -0,0 +1,49 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.dispensers; + +import io.nosqlbench.adapter.pulsar.PulsarSpace; +import io.nosqlbench.adapter.pulsar.ops.AdminNamespaceOp; +import io.nosqlbench.engine.api.activityimpl.uniform.DriverAdapter; +import io.nosqlbench.engine.api.templating.ParsedOp; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; +import org.apache.pulsar.client.admin.PulsarAdmin; + +import java.util.function.LongFunction; + +public class AdminNamespaceOpDispenser extends PulsarAdminOpDispenser { + + private final static Logger logger = LogManager.getLogger("AdminNamespaceOpDispenser"); + + public AdminNamespaceOpDispenser(DriverAdapter adapter, + ParsedOp op, + LongFunction tgtNameFunc, + PulsarSpace pulsarSpace) { + super(adapter, op, tgtNameFunc, pulsarSpace); + } + + @Override + public AdminNamespaceOp apply(long cycle) { + return new AdminNamespaceOp( + pulsarAdapterMetrics, + pulsarAdmin, + asyncApiFunc.apply(cycle), + adminDelOpFunc.apply(cycle), + tgtNameFunc.apply(cycle)); + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/AdminTenantOpDispenser.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/AdminTenantOpDispenser.java new file mode 100644 index 000000000..e19897ed2 --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/AdminTenantOpDispenser.java @@ -0,0 +1,57 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.dispensers; + +import io.nosqlbench.adapter.pulsar.PulsarSpace; +import io.nosqlbench.adapter.pulsar.ops.AdminTenantOp; +import io.nosqlbench.engine.api.activityimpl.uniform.DriverAdapter; +import io.nosqlbench.engine.api.templating.ParsedOp; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; +import org.apache.pulsar.client.admin.PulsarAdmin; + +import java.util.*; +import java.util.function.LongFunction; + +public class AdminTenantOpDispenser extends PulsarAdminOpDispenser { + + private final static Logger logger = LogManager.getLogger("AdminTenantOpDispenser"); + + private final LongFunction> adminRolesFunc; + private final LongFunction> allowedClustersFunc; + public AdminTenantOpDispenser(DriverAdapter adapter, + ParsedOp op, + LongFunction tgtNameFunc, + PulsarSpace pulsarSpace) { + super(adapter, op, tgtNameFunc, pulsarSpace); + + adminRolesFunc = lookupStaticStrSetOpValueFunc("admin_roles"); + allowedClustersFunc = lookupStaticStrSetOpValueFunc("allowed_clusters"); + } + + @Override + public AdminTenantOp apply(long cycle) { + return new AdminTenantOp( + pulsarAdapterMetrics, + pulsarAdmin, + asyncApiFunc.apply(cycle), + adminDelOpFunc.apply(cycle), + tgtNameFunc.apply(cycle), + adminRolesFunc.apply(cycle), + allowedClustersFunc.apply(cycle)); + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/AdminTopicOpDispenser.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/AdminTopicOpDispenser.java new file mode 100644 index 000000000..c92436128 --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/AdminTopicOpDispenser.java @@ -0,0 +1,60 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.dispensers; + +import io.nosqlbench.adapter.pulsar.PulsarSpace; +import io.nosqlbench.adapter.pulsar.ops.AdminTopicOp; +import io.nosqlbench.engine.api.activityimpl.uniform.DriverAdapter; +import io.nosqlbench.engine.api.templating.ParsedOp; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; +import org.apache.pulsar.client.admin.PulsarAdmin; + +import java.util.function.LongFunction; + +public class AdminTopicOpDispenser extends PulsarAdminOpDispenser { + + private final static Logger logger = LogManager.getLogger("AdminTopicOpDispenser"); + + private final LongFunction enablePartFunc; + private final LongFunction partNumFunc; + + public AdminTopicOpDispenser(DriverAdapter adapter, + ParsedOp op, + LongFunction tgtNameFunc, + PulsarSpace pulsarSpace) { + super(adapter, op, tgtNameFunc, pulsarSpace); + + // Non-partitioned topic is default + enablePartFunc = lookupStaticBoolConfigValueFunc("enable_partition", false); + partNumFunc = lookupStaticIntOpValueFunc("partition_num", 1); + } + + @Override + public AdminTopicOp apply(long cycle) { + + return new AdminTopicOp( + pulsarAdapterMetrics, + pulsarAdmin, + asyncApiFunc.apply(cycle), + adminDelOpFunc.apply(cycle), + tgtNameFunc.apply(cycle), + enablePartFunc.apply(cycle), + partNumFunc.apply(cycle) + ); + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/MessageConsumerOpDispenser.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/MessageConsumerOpDispenser.java new file mode 100644 index 000000000..82edc63a9 --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/MessageConsumerOpDispenser.java @@ -0,0 +1,105 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.dispensers; + +import io.nosqlbench.adapter.pulsar.PulsarSpace; +import io.nosqlbench.adapter.pulsar.ops.MessageConsumerOp; +import io.nosqlbench.adapter.pulsar.util.EndToEndStartingTimeSource; +import io.nosqlbench.adapter.pulsar.util.PulsarAdapterUtil; +import io.nosqlbench.adapter.pulsar.util.ReceivedMessageSequenceTracker; +import io.nosqlbench.engine.api.activityimpl.uniform.DriverAdapter; +import io.nosqlbench.engine.api.templating.ParsedOp; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; +import org.apache.pulsar.client.api.Consumer; + +import java.util.HashMap; +import java.util.Map; +import java.util.function.LongFunction; + +public class MessageConsumerOpDispenser extends PulsarClientOpDispenser { + + private final static Logger logger = LogManager.getLogger("MessageConsumerOpDispenser"); + + public static final String TOPIC_PATTERN_OP_PARAM = "topic_pattern"; + public static final String SUBSCRIPTION_NAME_OP_PARAM = "subscription_name"; + public static final String SUBSCRIPTION_TYPE_OP_PARAM = "subscription_type"; + public static final String CONSUMER_NAME_OP_PARAM = "consumer_name"; + public static final String RANGES_OP_PARAM = "ranges"; + + private final LongFunction topicPatternFunc; + private final LongFunction subscriptionNameFunc; + private final LongFunction subscriptionTypeFunc; + private final LongFunction cycleConsumerNameFunc; + private final LongFunction rangesFunc; + private final LongFunction e2eStartTimeSrcParamStrFunc; + private final LongFunction consumerFunction; + + private final ThreadLocal> receivedMessageSequenceTrackersForTopicThreadLocal = + ThreadLocal.withInitial(HashMap::new); + + public MessageConsumerOpDispenser(DriverAdapter adapter, + ParsedOp op, + LongFunction tgtNameFunc, + PulsarSpace pulsarSpace) { + super(adapter, op, tgtNameFunc, pulsarSpace); + + this.topicPatternFunc = lookupOptionalStrOpValueFunc(TOPIC_PATTERN_OP_PARAM); + this.subscriptionNameFunc = lookupMandtoryStrOpValueFunc(SUBSCRIPTION_NAME_OP_PARAM); + this.subscriptionTypeFunc = lookupOptionalStrOpValueFunc(SUBSCRIPTION_TYPE_OP_PARAM); + this.cycleConsumerNameFunc = lookupOptionalStrOpValueFunc(CONSUMER_NAME_OP_PARAM); + this.rangesFunc = lookupOptionalStrOpValueFunc(RANGES_OP_PARAM); + this.e2eStartTimeSrcParamStrFunc = lookupOptionalStrOpValueFunc( + PulsarAdapterUtil.DOC_LEVEL_PARAMS.E2E_STARTING_TIME_SOURCE.label, "none"); + this.consumerFunction = (l) -> getConsumer( + tgtNameFunc.apply(l), + topicPatternFunc.apply(l), + subscriptionNameFunc.apply(l), + subscriptionTypeFunc.apply(l), + cycleConsumerNameFunc.apply(l), + rangesFunc.apply(l)); + } + + @Override + public MessageConsumerOp apply(long cycle) { + return new MessageConsumerOp( + pulsarAdapterMetrics, + pulsarClient, + pulsarSchema, + asyncApiFunc.apply(cycle), + useTransactFunc.apply(cycle), + seqTrackingFunc.apply(cycle), + transactSupplierFunc.apply(cycle), + payloadRttFieldFunc.apply(cycle), + EndToEndStartingTimeSource.valueOf(e2eStartTimeSrcParamStrFunc.apply(cycle).toUpperCase()), + this::getReceivedMessageSequenceTracker, + consumerFunction.apply(cycle), + pulsarSpace.getPulsarNBClientConf().getConsumerTimeoutSeconds() + ); + } + + private ReceivedMessageSequenceTracker getReceivedMessageSequenceTracker(String topicName) { + return receivedMessageSequenceTrackersForTopicThreadLocal.get() + .computeIfAbsent(topicName, k -> createReceivedMessageSequenceTracker()); + } + + private ReceivedMessageSequenceTracker createReceivedMessageSequenceTracker() { + return new ReceivedMessageSequenceTracker(pulsarAdapterMetrics.getMsgErrOutOfSeqCounter(), + pulsarAdapterMetrics.getMsgErrDuplicateCounter(), + pulsarAdapterMetrics.getMsgErrLossCounter()); + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/MessageProducerOpDispenser.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/MessageProducerOpDispenser.java new file mode 100644 index 000000000..9f250e151 --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/MessageProducerOpDispenser.java @@ -0,0 +1,75 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.dispensers; + +import io.nosqlbench.adapter.pulsar.PulsarSpace; +import io.nosqlbench.adapter.pulsar.ops.MessageProducerOp; +import io.nosqlbench.engine.api.activityimpl.uniform.DriverAdapter; +import io.nosqlbench.engine.api.templating.ParsedOp; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; +import org.apache.pulsar.client.api.Producer; + +import java.util.Optional; +import java.util.function.LongFunction; + +public class MessageProducerOpDispenser extends PulsarClientOpDispenser { + + private final static Logger logger = LogManager.getLogger("MessageProducerOpDispenser"); + + public static final String PRODUCER_NAME_OP_PARAM = "producer_name"; + public static final String MSG_KEY_OP_PARAM = "msg_key"; + public static final String MSG_PROP_OP_PARAM = "msg_prop"; + public static final String MSG_VALUE_OP_PARAM = "msg_value"; + + private final LongFunction cycleProducerNameFunc; + private final LongFunction> producerFunc; + private final LongFunction msgKeyFunc; + private final LongFunction msgPropFunc; + private final LongFunction msgValueFunc; + + public MessageProducerOpDispenser(DriverAdapter adapter, + ParsedOp op, + LongFunction tgtNameFunc, + PulsarSpace pulsarSpace) { + super(adapter, op, tgtNameFunc, pulsarSpace); + + this.cycleProducerNameFunc = lookupOptionalStrOpValueFunc(PRODUCER_NAME_OP_PARAM); + this.producerFunc = (l) -> getProducer(tgtNameFunc.apply(l), cycleProducerNameFunc.apply(l)); + this.msgKeyFunc = lookupOptionalStrOpValueFunc(MSG_KEY_OP_PARAM); + this.msgPropFunc = lookupOptionalStrOpValueFunc(MSG_PROP_OP_PARAM); + this.msgValueFunc = lookupMandtoryStrOpValueFunc(MSG_VALUE_OP_PARAM); + } + + @Override + public MessageProducerOp apply(long cycle) { + return new MessageProducerOp( + pulsarAdapterMetrics, + pulsarClient, + pulsarSchema, + asyncApiFunc.apply(cycle), + useTransactFunc.apply(cycle), + seqTrackingFunc.apply(cycle), + transactSupplierFunc.apply(cycle), + errSimuTypeSetFunc.apply(cycle), + producerFunc.apply(cycle), + msgKeyFunc.apply(cycle), + msgPropFunc.apply(cycle), + msgValueFunc.apply(cycle) + ); + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/MessageReaderOpDispenser.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/MessageReaderOpDispenser.java new file mode 100644 index 000000000..f4e0a292a --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/MessageReaderOpDispenser.java @@ -0,0 +1,65 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.dispensers; + +import io.nosqlbench.adapter.pulsar.PulsarSpace; +import io.nosqlbench.adapter.pulsar.ops.MessageReaderOp; +import io.nosqlbench.adapter.pulsar.util.PulsarAdapterUtil; +import io.nosqlbench.engine.api.activityimpl.uniform.DriverAdapter; +import io.nosqlbench.engine.api.templating.ParsedOp; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; +import org.apache.pulsar.client.api.PulsarClient; +import org.apache.pulsar.client.api.Reader; +import org.apache.pulsar.client.api.Schema; + +import java.util.function.LongFunction; + +public class MessageReaderOpDispenser extends PulsarClientOpDispenser { + + private final static Logger logger = LogManager.getLogger("MessageReaderOpDispenser"); + + private final LongFunction cycleReaderNameFunc; + private final LongFunction msgStartPosStrFunc; + private final LongFunction readerFunc; + + public MessageReaderOpDispenser(DriverAdapter adapter, + ParsedOp op, + LongFunction tgtNameFunc, + PulsarSpace pulsarSpace) { + super(adapter, op, tgtNameFunc, pulsarSpace); + + this.cycleReaderNameFunc = lookupMandtoryStrOpValueFunc("reader_name"); + this.msgStartPosStrFunc = lookupOptionalStrOpValueFunc( + "start_msg_position", PulsarAdapterUtil.READER_MSG_POSITION_TYPE.earliest.label); + this.readerFunc = (l) -> getReader( + tgtNameFunc.apply(l), + cycleReaderNameFunc.apply(l), + msgStartPosStrFunc.apply(l)); + } + + @Override + public MessageReaderOp apply(long cycle) { + + return new MessageReaderOp( + pulsarAdapterMetrics, + pulsarClient, + pulsarSchema, + asyncApiFunc.apply(cycle), + readerFunc.apply(cycle)); + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/PulsarAdminOpDispenser.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/PulsarAdminOpDispenser.java new file mode 100644 index 000000000..a56752e29 --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/PulsarAdminOpDispenser.java @@ -0,0 +1,49 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.dispensers; + +import io.nosqlbench.adapter.pulsar.PulsarSpace; +import io.nosqlbench.adapter.pulsar.util.PulsarAdapterUtil; +import io.nosqlbench.engine.api.activityimpl.uniform.DriverAdapter; +import io.nosqlbench.engine.api.templating.ParsedOp; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; +import org.apache.pulsar.client.admin.PulsarAdmin; + +import java.util.function.LongFunction; + +public abstract class PulsarAdminOpDispenser extends PulsarBaseOpDispenser { + + private final static Logger logger = LogManager.getLogger("PulsarAdminOpDispenser"); + + + protected final PulsarAdmin pulsarAdmin; + protected final LongFunction adminDelOpFunc; + + public PulsarAdminOpDispenser(DriverAdapter adapter, + ParsedOp op, + LongFunction tgtNameFunc, + PulsarSpace pulsarSpace) { + super(adapter, op, tgtNameFunc, pulsarSpace); + + this.pulsarAdmin = pulsarSpace.getPulsarAdmin(); + + // Doc-level parameter: admin_delop + this.adminDelOpFunc = lookupStaticBoolConfigValueFunc( + PulsarAdapterUtil.DOC_LEVEL_PARAMS.ADMIN_DELOP.label, false); + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/PulsarBaseOpDispenser.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/PulsarBaseOpDispenser.java new file mode 100644 index 000000000..cfe9231ef --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/PulsarBaseOpDispenser.java @@ -0,0 +1,669 @@ +package io.nosqlbench.adapter.pulsar.dispensers; + +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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. + */ + +import io.nosqlbench.adapter.pulsar.PulsarSpace; +import io.nosqlbench.adapter.pulsar.exception.PulsarAdapterInvalidParamException; +import io.nosqlbench.adapter.pulsar.exception.PulsarAdapterUnexpectedException; +import io.nosqlbench.adapter.pulsar.ops.PulsarOp; +import io.nosqlbench.adapter.pulsar.util.PulsarAdapterMetrics; +import io.nosqlbench.adapter.pulsar.util.PulsarAdapterUtil; +import io.nosqlbench.api.config.NBNamedElement; +import io.nosqlbench.engine.api.activityimpl.BaseOpDispenser; +import io.nosqlbench.engine.api.activityimpl.uniform.DriverAdapter; +import io.nosqlbench.engine.api.templating.ParsedOp; +import org.apache.commons.lang3.BooleanUtils; +import org.apache.commons.lang3.StringUtils; +import org.apache.commons.lang3.math.NumberUtils; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; +import org.apache.pulsar.client.api.*; + +import java.util.*; +import java.util.function.LongFunction; +import java.util.function.Predicate; +import java.util.regex.Pattern; +import java.util.regex.PatternSyntaxException; +import java.util.stream.Collectors; + +public abstract class PulsarBaseOpDispenser extends BaseOpDispenser implements NBNamedElement { + + private final static Logger logger = LogManager.getLogger("PulsarBaseOpDispenser"); + + protected final ParsedOp parsedOp; + protected final PulsarSpace pulsarSpace; + protected final PulsarAdapterMetrics pulsarAdapterMetrics; + protected final LongFunction asyncApiFunc; + protected final LongFunction tgtNameFunc; + + protected final int totalThreadNum; + + protected final long totalCycleNum; + + public PulsarBaseOpDispenser(DriverAdapter adapter, + ParsedOp op, + LongFunction tgtNameFunc, + PulsarSpace pulsarSpace) { + + super(adapter, op); + + this.parsedOp = op; + this.tgtNameFunc = tgtNameFunc; + this.pulsarSpace = pulsarSpace; + + // Doc-level parameter: async_api + this.asyncApiFunc = lookupStaticBoolConfigValueFunc( + PulsarAdapterUtil.DOC_LEVEL_PARAMS.ASYNC_API.label, true); + + String defaultMetricsPrefix = getDefaultMetricsPrefix(this.parsedOp); + this.pulsarAdapterMetrics = new PulsarAdapterMetrics(this, defaultMetricsPrefix); + pulsarAdapterMetrics.initPulsarAdapterInstrumentation(); + + totalThreadNum = NumberUtils.toInt(parsedOp.getStaticValue("threads")); + totalCycleNum = NumberUtils.toLong(parsedOp.getStaticValue("cycles")); + } + + @Override + public String getName() { + return "PulsarBaseOpDispenser"; + } + + public PulsarSpace getPulsarSpace() { return pulsarSpace; } + + protected LongFunction lookupStaticBoolConfigValueFunc(String paramName, boolean defaultValue) { + LongFunction booleanLongFunction; + booleanLongFunction = (l) -> parsedOp.getOptionalStaticConfig(paramName, String.class) + .filter(Predicate.not(String::isEmpty)) + .map(value -> BooleanUtils.toBoolean(value)) + .orElse(defaultValue); + logger.info("{}: {}", paramName, booleanLongFunction.apply(0)); + return booleanLongFunction; + } + + protected LongFunction> lookupStaticStrSetOpValueFunc(String paramName) { + LongFunction> setStringLongFunction; + setStringLongFunction = (l) -> parsedOp.getOptionalStaticValue(paramName, String.class) + .filter(Predicate.not(String::isEmpty)) + .map(value -> { + Set set = new HashSet<>(); + + if (StringUtils.contains(value,',')) { + set = Arrays.stream(value.split(",")) + .map(String::trim) + .filter(Predicate.not(String::isEmpty)) + .collect(Collectors.toCollection(LinkedHashSet::new)); + } + + return set; + }).orElse(Collections.emptySet()); + logger.info("{}: {}", paramName, setStringLongFunction.apply(0)); + return setStringLongFunction; + } + + // If the corresponding Op parameter is not provided, use the specified default value + protected LongFunction lookupStaticIntOpValueFunc(String paramName, int defaultValue) { + LongFunction integerLongFunction; + integerLongFunction = (l) -> parsedOp.getOptionalStaticValue(paramName, String.class) + .filter(Predicate.not(String::isEmpty)) + .map(value -> NumberUtils.toInt(value)) + .map(value -> { + if (value < 0) return 0; + else return value; + }).orElse(defaultValue); + logger.info("{}: {}", paramName, integerLongFunction.apply(0)); + return integerLongFunction; + } + + // If the corresponding Op parameter is not provided, use the specified default value + protected LongFunction lookupOptionalStrOpValueFunc(String paramName, String defaultValue) { + LongFunction stringLongFunction; + stringLongFunction = parsedOp.getAsOptionalFunction(paramName, String.class) + .orElse((l) -> defaultValue); + logger.info("{}: {}", paramName, stringLongFunction.apply(0)); + + return stringLongFunction; + } + protected LongFunction lookupOptionalStrOpValueFunc(String paramName) { + return lookupOptionalStrOpValueFunc(paramName, ""); + } + + // Mandatory Op parameter. Throw an error if not specified or having empty value + protected LongFunction lookupMandtoryStrOpValueFunc(String paramName) { + LongFunction stringLongFunction; + stringLongFunction = parsedOp.getAsRequiredFunction(paramName, String.class); + logger.info("{}: {}", paramName, stringLongFunction.apply(0)); + + return stringLongFunction; + } + + /** + * Get a proper Pulsar API metrics prefix depending on the API type + * + * @param apiType - Pulsar API type: producer, consumer, reader, etc. + * @param apiObjName - actual name of a producer, a consumer, a reader, etc. + * @param topicName - topic name + * @return String + */ + private String getPulsarAPIMetricsPrefix(String apiType, String apiObjName, String topicName) { + String apiMetricsPrefix = ""; + + if (PulsarAdapterUtil.isValidPulsarApiType(apiType)) { + if (!StringUtils.isBlank(apiObjName)) { + apiMetricsPrefix = apiObjName + "_"; + } else { + // we want a meaningful name for the API object (producer, consumer, reader, etc.) + // we are not appending the topic name + apiMetricsPrefix = apiType; + + if (apiType.equalsIgnoreCase(PulsarAdapterUtil.PULSAR_API_TYPE.PRODUCER.label)) + apiMetricsPrefix += pulsarSpace.getProducerSetCnt(); + else if (apiType.equalsIgnoreCase(PulsarAdapterUtil.PULSAR_API_TYPE.CONSUMER.label)) + apiMetricsPrefix += pulsarSpace.getConsumerSetCnt(); + else if (apiType.equalsIgnoreCase(PulsarAdapterUtil.PULSAR_API_TYPE.READER.label)) + apiMetricsPrefix += pulsarSpace.getReaderSetCnt(); + + apiMetricsPrefix += "_"; + } + + apiMetricsPrefix += topicName + "_"; + apiMetricsPrefix = apiMetricsPrefix + // default name for tests/demos (in all Pulsar examples) is persistent://public/default/test -> use just the topic name test + .replace("persistent://public/default/", "") + // always remove topic type + .replace("non-persistent://", "") + .replace("persistent://", "") + // persistent://tenant/namespace/topicname -> tenant_namespace_topicname + .replace("/", "_"); + + apiMetricsPrefix += "--"; + } + + return apiMetricsPrefix; + } + + ////////////////////////////////////// + // Producer Processing --> start + ////////////////////////////////////// + // + + // Topic name IS mandatory for a producer + // - It must be set at either global level or cycle level + // - If set at both levels, cycle level setting takes precedence + private String getEffectiveProducerTopicName(String cycleTopicName) { + if (!StringUtils.isBlank(cycleTopicName)) { + return cycleTopicName; + } + + String globalTopicName = pulsarSpace.getPulsarNBClientConf().getProducerTopicName(); + if (!StringUtils.isBlank(globalTopicName)) { + return globalTopicName; + } + + throw new PulsarAdapterInvalidParamException( + "Effective topic name for a producer can't NOT be empty, " + + "it must be set either as a corresponding adapter Op parameter value or " + + "set in the global Pulsar conf file."); + } + + // Producer name is NOT mandatory + // - It can be set at either global level or cycle level + // - If set at both levels, cycle level setting takes precedence + private String getEffectiveProducerName(String cycleProducerName) { + if (!StringUtils.isBlank(cycleProducerName)) { + return cycleProducerName; + } + + String globalProducerName = pulsarSpace.getPulsarNBClientConf().getProducerName(); + if (!StringUtils.isBlank(globalProducerName)) { + return globalProducerName; + } + + return ""; + } + + public Producer getProducer(String cycleTopicName, String cycleProducerName) { + String topicName = getEffectiveProducerTopicName(cycleTopicName); + String producerName = getEffectiveProducerName(cycleProducerName); + + String producerCacheKey = PulsarAdapterUtil.buildCacheKey(producerName, topicName); + Producer producer = pulsarSpace.getProducer(producerCacheKey); + + if (producer == null) { + PulsarClient pulsarClient = pulsarSpace.getPulsarClient(); + + // Get other possible producer settings that are set at global level + Map producerConf = pulsarSpace.getPulsarNBClientConf().getProducerConfMapTgt(); + + // Remove global level settings: "topicName" and "producerName" + producerConf.remove(PulsarAdapterUtil.PRODUCER_CONF_STD_KEY.topicName.label); + producerConf.remove(PulsarAdapterUtil.PRODUCER_CONF_STD_KEY.producerName.label); + + try { + ProducerBuilder producerBuilder = pulsarClient. + newProducer(pulsarSpace.getPulsarSchema()). + loadConf(producerConf). + topic(topicName); + + if (!StringUtils.isAnyBlank(producerName)) { + producerBuilder = producerBuilder.producerName(producerName); + } + + producer = producerBuilder.create(); + pulsarSpace.setProducer(producerCacheKey, producer); + + pulsarAdapterMetrics.registerProducerApiMetrics(producer, + getPulsarAPIMetricsPrefix( + PulsarAdapterUtil.PULSAR_API_TYPE.PRODUCER.label, + producerName, + topicName)); + } + catch (PulsarClientException ple) { + throw new PulsarAdapterUnexpectedException("Failed to create a Pulsar producer."); + } + } + + return producer; + } + + // + ////////////////////////////////////// + // Producer Processing <-- end + ////////////////////////////////////// + + + + ////////////////////////////////////// + // Consumer Processing --> start + ////////////////////////////////////// + // + + private String getEffectiveConsumerTopicNameListStr(String cycleTopicNameListStr) { + if (!StringUtils.isBlank(cycleTopicNameListStr)) { + return cycleTopicNameListStr; + } + + String globalTopicNames = pulsarSpace.getPulsarNBClientConf().getConsumerTopicNames(); + if (!StringUtils.isBlank(globalTopicNames)) { + return globalTopicNames; + } + + return ""; + } + + private List getEffectiveConsumerTopicNameList(String cycleTopicNameListStr) { + String effectiveTopicNamesStr = getEffectiveConsumerTopicNameListStr(cycleTopicNameListStr); + + String[] names = effectiveTopicNamesStr.split("[;,]"); + ArrayList effectiveTopicNameList = new ArrayList<>(); + + for (String name : names) { + if (!StringUtils.isBlank(name)) + effectiveTopicNameList.add(name.trim()); + } + + return effectiveTopicNameList; + } + + private String getEffectiveConsumerTopicPatternStr(String cycleTopicPatternStr) { + if (!StringUtils.isBlank(cycleTopicPatternStr)) { + return cycleTopicPatternStr; + } + + String globalTopicsPattern = pulsarSpace.getPulsarNBClientConf().getConsumerTopicPattern(); + if (!StringUtils.isBlank(globalTopicsPattern)) { + return globalTopicsPattern; + } + + return ""; + } + + private Pattern getEffectiveConsumerTopicPattern(String cycleTopicPatternStr) { + String effectiveTopicPatternStr = getEffectiveConsumerTopicPatternStr(cycleTopicPatternStr); + Pattern topicsPattern; + try { + if (!StringUtils.isBlank(effectiveTopicPatternStr)) + topicsPattern = Pattern.compile(effectiveTopicPatternStr); + else + topicsPattern = null; + } catch (PatternSyntaxException pse) { + topicsPattern = null; + } + return topicsPattern; + } + + + // Subscription name is NOT mandatory + // - It can be set at either global level or cycle level + // - If set at both levels, cycle level setting takes precedence + private String getEffectiveSubscriptionName(String cycleSubscriptionName) { + if (!StringUtils.isBlank(cycleSubscriptionName)) { + return cycleSubscriptionName; + } + + String globalSubscriptionName = pulsarSpace.getPulsarNBClientConf().getConsumerSubscriptionName(); + if (!StringUtils.isBlank(globalSubscriptionName)) { + return globalSubscriptionName; + } + + throw new PulsarAdapterInvalidParamException( + "Effective subscription name for a consumer can't NOT be empty, " + + "it must be set either as a corresponding adapter Op parameter value or " + + "set in the global Pulsar conf file."); + } + + private String getEffectiveSubscriptionTypeStr(String cycleSubscriptionType) { + String subscriptionTypeStr = ""; + + if (!StringUtils.isBlank(cycleSubscriptionType)) { + subscriptionTypeStr = cycleSubscriptionType; + } + else { + String globalSubscriptionType = pulsarSpace.getPulsarNBClientConf().getConsumerSubscriptionType(); + if (!StringUtils.isBlank(globalSubscriptionType)) { + subscriptionTypeStr = globalSubscriptionType; + } + } + + if (StringUtils.isNotBlank(subscriptionTypeStr) && + !PulsarAdapterUtil.isValidSubscriptionType(subscriptionTypeStr)) { + throw new PulsarAdapterInvalidParamException( + "Invalid effective subscription type for a consumer (\"" + subscriptionTypeStr + "\"). " + + "It must be one of the following values: " + PulsarAdapterUtil.getValidSubscriptionTypeList()); + } + + return subscriptionTypeStr; + } + private SubscriptionType getEffectiveSubscriptionType(String cycleSubscriptionType) { + String effectiveSubscriptionStr = getEffectiveSubscriptionTypeStr(cycleSubscriptionType); + SubscriptionType subscriptionType = SubscriptionType.Exclusive; // default subscription type + + if (!StringUtils.isBlank(effectiveSubscriptionStr)) { + subscriptionType = SubscriptionType.valueOf(effectiveSubscriptionStr); + } + + return subscriptionType; + } + + private String getEffectiveConsumerName(String cycleConsumerName) { + if (!StringUtils.isBlank(cycleConsumerName)) { + return cycleConsumerName; + } + + String globalConsumerName = pulsarSpace.getPulsarNBClientConf().getConsumerName(); + if (!StringUtils.isBlank(globalConsumerName)) { + return globalConsumerName; + } + + return ""; + } + + public Consumer getConsumer(String cycleTopicNameListStr, + String cycleTopicPatternStr, + String cycleSubscriptionName, + String cycleSubscriptionType, + String cycleConsumerName, + String cycleKeySharedSubscriptionRanges) { + + List topicNameList = getEffectiveConsumerTopicNameList(cycleTopicNameListStr); + String topicPatternStr = getEffectiveConsumerTopicPatternStr(cycleTopicPatternStr); + Pattern topicPattern = getEffectiveConsumerTopicPattern(cycleTopicPatternStr); + String subscriptionName = getEffectiveSubscriptionName(cycleSubscriptionName); + SubscriptionType subscriptionType = getEffectiveSubscriptionType(cycleSubscriptionType); + String consumerName = getEffectiveConsumerName(cycleConsumerName); + + if ( subscriptionType.equals(SubscriptionType.Exclusive) && (totalThreadNum > 1) ) { + throw new PulsarAdapterInvalidParamException( + MessageConsumerOpDispenser.SUBSCRIPTION_TYPE_OP_PARAM, + "creating multiple consumers of \"Exclusive\" subscription type under the same subscription name"); + } + + if ( (topicNameList.isEmpty() && (topicPattern == null)) || + (!topicNameList.isEmpty() && (topicPattern != null)) ) { + throw new PulsarAdapterInvalidParamException( + "Invalid combination of topic name(s) and topic patterns; only specify one parameter!"); + } + + boolean multiTopicConsumer = (topicNameList.size() > 1 || (topicPattern != null)); + + String consumerTopicListString; + if (!topicNameList.isEmpty()) { + consumerTopicListString = String.join("|", topicNameList); + } else { + consumerTopicListString = topicPatternStr; + } + + String consumerCacheKey = PulsarAdapterUtil.buildCacheKey( + consumerName, + subscriptionName, + consumerTopicListString); + Consumer consumer = pulsarSpace.getConsumer(consumerCacheKey); + + if (consumer == null) { + PulsarClient pulsarClient = pulsarSpace.getPulsarClient(); + + // Get other possible consumer settings that are set at global level + Map consumerConf = new HashMap<>(pulsarSpace.getPulsarNBClientConf().getConsumerConfMapTgt()); + + // Remove global level settings: + // - "topicNames", "topicsPattern", "subscriptionName", "subscriptionType", "consumerName" + consumerConf.remove(PulsarAdapterUtil.CONSUMER_CONF_STD_KEY.topicNames.label); + consumerConf.remove(PulsarAdapterUtil.CONSUMER_CONF_STD_KEY.topicsPattern.label); + consumerConf.remove(PulsarAdapterUtil.CONSUMER_CONF_STD_KEY.subscriptionName.label); + consumerConf.remove(PulsarAdapterUtil.CONSUMER_CONF_STD_KEY.subscriptionType.label); + consumerConf.remove(PulsarAdapterUtil.CONSUMER_CONF_STD_KEY.consumerName.label); + // Remove non-standard consumer configuration properties + consumerConf.remove(PulsarAdapterUtil.CONSUMER_CONF_CUSTOM_KEY.timeout.label); + + try { + ConsumerBuilder consumerBuilder; + + if (!multiTopicConsumer) { + assert (topicNameList.size() == 1); + consumerBuilder = pulsarClient.newConsumer(pulsarSpace.getPulsarSchema()); + consumerBuilder.topic(topicNameList.get(0)); + } + else { + consumerBuilder = pulsarClient.newConsumer(); + if (!topicNameList.isEmpty()) { + assert (topicNameList.size() > 1); + consumerBuilder.topics(topicNameList); + } + else { + consumerBuilder.topicsPattern(topicPattern); + } + } + + consumerBuilder. + loadConf(consumerConf). + subscriptionName(subscriptionName). + subscriptionType(subscriptionType); + + if (!StringUtils.isBlank(consumerName)) + consumerBuilder.consumerName(consumerName); + + if (subscriptionType == SubscriptionType.Key_Shared) { + KeySharedPolicy keySharedPolicy = KeySharedPolicy.autoSplitHashRange(); + if (cycleKeySharedSubscriptionRanges != null && !cycleKeySharedSubscriptionRanges.isEmpty()) { + Range[] ranges = parseRanges(cycleKeySharedSubscriptionRanges); + logger.info("Configuring KeySharedPolicy#stickyHashRange with ranges {}", ranges); + keySharedPolicy = KeySharedPolicy.stickyHashRange().ranges(ranges); + } + consumerBuilder.keySharedPolicy(keySharedPolicy); + } + + consumer = consumerBuilder.subscribe(); + pulsarSpace.setConsumer(consumerCacheKey, consumer); + + if (instrument) { + pulsarAdapterMetrics.registerConsumerApiMetrics( + consumer, + getPulsarAPIMetricsPrefix( + PulsarAdapterUtil.PULSAR_API_TYPE.CONSUMER.label, + consumerName, + consumerTopicListString)); + } + + } + catch (PulsarClientException ple) { + throw new PulsarAdapterUnexpectedException("Failed to create a Pulsar consumer!"); + } + } + + return consumer; + } + + private static Range[] parseRanges(String ranges) { + if (ranges == null || ranges.isEmpty()) { + return new Range[0]; + } + String[] split = ranges.split(","); + Range[] result = new Range[split.length]; + for (int i = 0; i < split.length; i++) { + String range = split[i]; + int pos = range.indexOf(".."); + if (pos <= 0) { + throw new IllegalArgumentException("Invalid range '" + range + "'"); + } + try { + int start = Integer.parseInt(range.substring(0, pos)); + int end = Integer.parseInt(range.substring(pos + 2)); + result[i] = Range.of(start, end); + } catch (NumberFormatException err) { + throw new IllegalArgumentException("Invalid range '" + range + "'"); + } + } + return result; + } + + // + ////////////////////////////////////// + // Consumer Processing <-- end + ////////////////////////////////////// + + + + ////////////////////////////////////// + // Reader Processing --> Start + ////////////////////////////////////// + // + + // Topic name IS mandatory for a reader + // - It must be set at either global level or cycle level + // - If set at both levels, cycle level setting takes precedence + private String getEffectiveReaderTopicName(String cycleReaderTopicName) { + if (!StringUtils.isBlank(cycleReaderTopicName)) { + return cycleReaderTopicName; + } + + String globalReaderTopicName = pulsarSpace.getPulsarNBClientConf().getReaderTopicName(); + if (!StringUtils.isBlank(globalReaderTopicName)) { + return globalReaderTopicName; + } + + throw new PulsarAdapterInvalidParamException( + "Effective topic name for a reader can't NOT be empty, " + + "it must be set either as a corresponding adapter Op parameter value or " + + "set in the global Pulsar conf file."); + } + + // Reader name is NOT mandatory + // - It can be set at either global level or cycle level + // - If set at both levels, cycle level setting takes precedence + private String getEffectiveReaderName(String cycleReaderName) { + if (!StringUtils.isBlank(cycleReaderName)) { + return cycleReaderName; + } + + String globalReaderName = pulsarSpace.getPulsarNBClientConf().getReaderName(); + if (!StringUtils.isBlank(globalReaderName)) { + return globalReaderName; + } + + return ""; + } + + private String getEffectiveStartMsgPosStr(String cycleStartMsgPosStr) { + if (!StringUtils.isBlank(cycleStartMsgPosStr)) { + return cycleStartMsgPosStr; + } + + String globalStartMsgPosStr = pulsarSpace.getPulsarNBClientConf().getStartMsgPosStr(); + if (!StringUtils.isBlank(globalStartMsgPosStr)) { + return globalStartMsgPosStr; + } + + return PulsarAdapterUtil.READER_MSG_POSITION_TYPE.latest.label; + } + + public Reader getReader(String cycleTopicName, + String cycleReaderName, + String cycleStartMsgPos) { + + String topicName = getEffectiveReaderTopicName(cycleTopicName); + String readerName = getEffectiveReaderName(cycleReaderName); + String startMsgPosStr = getEffectiveStartMsgPosStr(cycleStartMsgPos); + if (!PulsarAdapterUtil.isValideReaderStartPosition(startMsgPosStr)) { + throw new RuntimeException("Reader:: Invalid value for reader start message position!"); + } + + String readerCacheKey = PulsarAdapterUtil.buildCacheKey(topicName, readerName, startMsgPosStr); + Reader reader = pulsarSpace.getReader(readerCacheKey); + + if (reader == null) { + PulsarClient pulsarClient = pulsarSpace.getPulsarClient();; + + Map readerConf = pulsarSpace.getPulsarNBClientConf().getReaderConfMapTgt(); + + // Remove global level settings: "topicName" and "readerName" + readerConf.remove(PulsarAdapterUtil.READER_CONF_STD_KEY.topicName.label); + readerConf.remove(PulsarAdapterUtil.READER_CONF_STD_KEY.readerName.label); + // Remove non-standard reader configuration properties + readerConf.remove(PulsarAdapterUtil.READER_CONF_CUSTOM_KEY.startMessagePos.label); + + try { + ReaderBuilder readerBuilder = pulsarClient. + newReader(pulsarSpace.getPulsarSchema()). + loadConf(readerConf). + topic(topicName). + readerName(readerName); + + MessageId startMsgId = MessageId.latest; + if (startMsgPosStr.equalsIgnoreCase(PulsarAdapterUtil.READER_MSG_POSITION_TYPE.earliest.label)) { + startMsgId = MessageId.earliest; + } + //TODO: custom start message position is NOT supported yet + //else if (startMsgPosStr.startsWith(PulsarAdapterUtil.READER_MSG_POSITION_TYPE.custom.label)) { + // startMsgId = MessageId.latest; + //} + + reader = readerBuilder.startMessageId(startMsgId).create(); + + } catch (PulsarClientException ple) { + ple.printStackTrace(); + throw new RuntimeException("Unable to create a Pulsar reader!"); + } + + pulsarSpace.setReader(readerCacheKey, reader); + } + + return reader; + } + // + ////////////////////////////////////// + // Reader Processing <-- end + ////////////////////////////////////// +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/PulsarClientOpDispenser.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/PulsarClientOpDispenser.java new file mode 100644 index 000000000..b61dcbf33 --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/dispensers/PulsarClientOpDispenser.java @@ -0,0 +1,124 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.dispensers; + +import com.codahale.metrics.Timer; +import io.nosqlbench.adapter.pulsar.PulsarSpace; +import io.nosqlbench.adapter.pulsar.util.PulsarAdapterUtil; +import io.nosqlbench.engine.api.activityimpl.uniform.DriverAdapter; +import io.nosqlbench.engine.api.templating.ParsedOp; +import org.apache.commons.lang3.StringUtils; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; +import org.apache.pulsar.client.api.PulsarClient; +import org.apache.pulsar.client.api.PulsarClientException; +import org.apache.pulsar.client.api.Schema; +import org.apache.pulsar.client.api.transaction.Transaction; + +import java.util.*; +import java.util.concurrent.ExecutionException; +import java.util.function.LongFunction; +import java.util.function.Predicate; +import java.util.function.Supplier; +import java.util.stream.Collectors; + +public abstract class PulsarClientOpDispenser extends PulsarBaseOpDispenser { + + private final static Logger logger = LogManager.getLogger("PulsarClientOpDispenser"); + + protected final PulsarClient pulsarClient; + protected final Schema pulsarSchema; + + protected final LongFunction useTransactFunc; + // TODO: add support for "operation number per transaction" + // protected final LongFunction transactBatchNumFunc; + protected final LongFunction seqTrackingFunc; + protected final LongFunction payloadRttFieldFunc; + protected final LongFunction> transactSupplierFunc; + protected final LongFunction> errSimuTypeSetFunc; + + public PulsarClientOpDispenser(DriverAdapter adapter, + ParsedOp op, + LongFunction tgtNameFunc, + PulsarSpace pulsarSpace) { + super(adapter, op, tgtNameFunc, pulsarSpace); + + this.pulsarClient = pulsarSpace.getPulsarClient(); + this.pulsarSchema = pulsarSpace.getPulsarSchema(); + + // Doc-level parameter: use_transaction + this.useTransactFunc = lookupStaticBoolConfigValueFunc( + PulsarAdapterUtil.DOC_LEVEL_PARAMS.USE_TRANSACTION.label, false); + + // TODO: add support for "operation number per transaction" + // Doc-level parameter: transact_batch_num + // this.transactBatchNumFunc = lookupStaticIntOpValueFunc( + // PulsarAdapterUtil.DOC_LEVEL_PARAMS.TRANSACT_BATCH_NUM.label, 1); + + // Doc-level parameter: seq_tracking + this.seqTrackingFunc = lookupStaticBoolConfigValueFunc( + PulsarAdapterUtil.DOC_LEVEL_PARAMS.SEQ_TRACKING.label, false); + + // Doc-level parameter: payload-tracking-field + this.payloadRttFieldFunc = (l) -> parsedOp.getStaticConfigOr( + PulsarAdapterUtil.DOC_LEVEL_PARAMS.RTT_TRACKING_FIELD.label, ""); + + this.transactSupplierFunc = (l) -> getTransactionSupplier(); + + this.errSimuTypeSetFunc = getStaticErrSimuTypeSetOpValueFunc(); + } + + protected Supplier getTransactionSupplier() { + return () -> { + try (Timer.Context time = pulsarAdapterMetrics.getCommitTransactionTimer().time() ){ + return pulsarClient + .newTransaction() + .build() + .get(); + } catch (ExecutionException | InterruptedException err) { + if (logger.isWarnEnabled()) { + logger.warn("Error while starting a new transaction", err); + } + throw new RuntimeException(err); + } catch (PulsarClientException err) { + throw new RuntimeException("Transactions are not enabled on Pulsar Client, " + + "please set client.enableTransaction=true in your Pulsar Client configuration"); + } + }; + } + + protected LongFunction> getStaticErrSimuTypeSetOpValueFunc() { + LongFunction> setStringLongFunction; + setStringLongFunction = (l) -> parsedOp.getOptionalStaticValue("seqerr_simu", String.class) + .filter(Predicate.not(String::isEmpty)) + .map(value -> { + Set set = new HashSet<>(); + + if (StringUtils.contains(value,',')) { + set = Arrays.stream(value.split(",")) + .map(PulsarAdapterUtil.SEQ_ERROR_SIMU_TYPE::parseSimuType) + .filter(Optional::isPresent) + .map(Optional::get) + .collect(Collectors.toCollection(LinkedHashSet::new)); + } + + return set; + }).orElse(Collections.emptySet()); + logger.info("seqerr_simu: {}", setStringLongFunction.apply(0)); + return setStringLongFunction; + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/exception/PulsarAdapterAsyncOperationFailedException.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/exception/PulsarAdapterAsyncOperationFailedException.java new file mode 100644 index 000000000..3cb31e4ba --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/exception/PulsarAdapterAsyncOperationFailedException.java @@ -0,0 +1,26 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.exception; + +public class PulsarAdapterAsyncOperationFailedException extends RuntimeException { + + public PulsarAdapterAsyncOperationFailedException(Throwable t) { + super(t); + printStackTrace(); + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/exception/PulsarAdapterInvalidParamException.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/exception/PulsarAdapterInvalidParamException.java new file mode 100644 index 000000000..5996e8573 --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/exception/PulsarAdapterInvalidParamException.java @@ -0,0 +1,29 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.exception; + +public class PulsarAdapterInvalidParamException extends RuntimeException { + + public PulsarAdapterInvalidParamException(String paramName, String errDesc) { + super("Invalid setting for parameter (" + paramName + "): " + errDesc); + } + + public PulsarAdapterInvalidParamException(String fullErrDesc) { + super(fullErrDesc); + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/exception/PulsarAdapterUnexpectedException.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/exception/PulsarAdapterUnexpectedException.java new file mode 100644 index 000000000..4f0031fce --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/exception/PulsarAdapterUnexpectedException.java @@ -0,0 +1,30 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.exception; + +public class PulsarAdapterUnexpectedException extends RuntimeException { + + public PulsarAdapterUnexpectedException(String message) { + super(message); + printStackTrace(); + } + public PulsarAdapterUnexpectedException(Exception e) { + super(e); + printStackTrace(); + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/exception/PulsarAdapterUnsupportedOpException.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/exception/PulsarAdapterUnsupportedOpException.java new file mode 100644 index 000000000..475d358ea --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/exception/PulsarAdapterUnsupportedOpException.java @@ -0,0 +1,25 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.exception; + +public class PulsarAdapterUnsupportedOpException extends RuntimeException { + + public PulsarAdapterUnsupportedOpException(String pulsarOpType) { + super("Unsupported Pulsar adapter operation type: \"" + pulsarOpType + "\""); + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/AdminNamespaceOp.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/AdminNamespaceOp.java new file mode 100644 index 000000000..2caca86bc --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/AdminNamespaceOp.java @@ -0,0 +1,122 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.ops; + +import io.nosqlbench.adapter.pulsar.exception.PulsarAdapterUnexpectedException; +import io.nosqlbench.adapter.pulsar.util.PulsarAdapterMetrics; +import org.apache.commons.lang3.StringUtils; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; +import org.apache.pulsar.client.admin.Namespaces; +import org.apache.pulsar.client.admin.PulsarAdmin; +import org.apache.pulsar.client.admin.PulsarAdminException; + +import java.util.concurrent.CompletableFuture; + +public class AdminNamespaceOp extends PulsarAdminOp { + + private final static Logger logger = LogManager.getLogger(AdminNamespaceOp.class); + + // in format: / + private final String nsName; + + public AdminNamespaceOp(PulsarAdapterMetrics pulsarAdapterMetrics, + PulsarAdmin pulsarAdmin, + boolean asyncApi, + boolean adminDelOp, + String nsName) { + super(pulsarAdapterMetrics, pulsarAdmin, asyncApi, adminDelOp); + this.nsName = nsName; + } + + @Override + public Void apply(long value) { + + // Do nothing if the namespace name is empty + if ( !StringUtils.isBlank(nsName) ) { + + Namespaces namespaces = pulsarAdmin.namespaces(); + + // Admin API - create tenants and namespaces + if (!adminDelOp) { + try { + if (!asyncApi) { + namespaces.createNamespace(nsName); + if (logger.isDebugEnabled()) { + logger.debug("Successful sync creation of namespace \"{}\"", nsName); + } + } else { + CompletableFuture future = namespaces.createNamespaceAsync(nsName); + future.whenComplete((unused, throwable) -> { + if (logger.isDebugEnabled()) { + logger.trace("Successful async creation of namespace \"{}\"", nsName); + } + }).exceptionally(ex -> { + if (logger.isDebugEnabled()) { + logger.error("Failed async creation of namespace \"{}\"", nsName); + } + return null; + }); + } + } + catch (PulsarAdminException.ConflictException ce) { + if (logger.isDebugEnabled()) { + logger.error("Namespace \"{}\" already exists - skip creation!", nsName); + } + } + catch (PulsarAdminException e) { + throw new PulsarAdapterUnexpectedException( + "Unexpected error when creating pulsar namespace \"" + nsName + "\""); + } + } + // Admin API - delete tenants and namespaces + else { + try { + if (!asyncApi) { + namespaces.deleteNamespace(nsName); + if (logger.isDebugEnabled()) { + logger.debug("Successful sync deletion of namespace \"{}\"", nsName); + } + } else { + CompletableFuture future = namespaces.deleteNamespaceAsync(nsName, true); + future.whenComplete((unused, throwable) -> { + if (logger.isDebugEnabled()) { + logger.debug("Successful sync deletion of namespace \"{}\"", nsName); + } + }).exceptionally(ex -> { + if (logger.isDebugEnabled()) { + logger.error("Failed async deletion of namespace \"{}\"", nsName); + } + return null; + }); + } + } + catch (PulsarAdminException.NotFoundException nfe) { + if (logger.isDebugEnabled()) { + logger.error("Namespace \"{}\" doesn't exists - skip deletion!", nsName); + } + } + catch (PulsarAdminException e) { + throw new PulsarAdapterUnexpectedException( + "Unexpected error when deleting pulsar namespace \"" + nsName + "\""); + } + } + } + + return null; + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/AdminTenantOp.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/AdminTenantOp.java new file mode 100644 index 000000000..f3050e82a --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/AdminTenantOp.java @@ -0,0 +1,147 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.ops; + +import io.nosqlbench.adapter.pulsar.PulsarDriverAdapter; +import io.nosqlbench.adapter.pulsar.exception.PulsarAdapterUnexpectedException; +import io.nosqlbench.adapter.pulsar.util.PulsarAdapterMetrics; +import org.apache.commons.collections.CollectionUtils; +import org.apache.commons.lang3.StringUtils; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; +import org.apache.pulsar.client.admin.*; +import org.apache.pulsar.common.policies.data.TenantInfo; + +import java.util.HashSet; +import java.util.Set; +import java.util.concurrent.CompletableFuture; + +public class AdminTenantOp extends PulsarAdminOp { + + private final static Logger logger = LogManager.getLogger(AdminTenantOp.class); + + private final Set adminRoles; + private final Set allowedClusters; + private final String tntName; + + public AdminTenantOp(PulsarAdapterMetrics pulsarAdapterMetrics, + PulsarAdmin pulsarAdmin, + boolean asyncApi, + boolean adminDelOp, + String tntName, + Set adminRoles, + Set allowedClusters) { + super(pulsarAdapterMetrics, pulsarAdmin, asyncApi, adminDelOp); + + this.tntName = tntName; + this.adminRoles = adminRoles; + this.allowedClusters = allowedClusters; + } + + @Override + public Void apply(long value) { + + // Do nothing if the tenant name is empty + if ( !StringUtils.isBlank(tntName) ) { + Tenants tenants = pulsarAdmin.tenants(); + + // Admin API - create tenants and namespaces + if (!adminDelOp) { + try { + Set existingPulsarClusters = new HashSet<>(); + Clusters clusters = pulsarAdmin.clusters(); + CollectionUtils.addAll(existingPulsarClusters, clusters.getClusters().listIterator()); + + TenantInfo tenantInfo = TenantInfo.builder() + .adminRoles(adminRoles) + .allowedClusters(!allowedClusters.isEmpty() ? allowedClusters : existingPulsarClusters) + .build(); + + if (!asyncApi) { + tenants.createTenant(tntName, tenantInfo); + if (logger.isDebugEnabled()) { + logger.debug("Successful sync creation of tenant \"{}\"", tntName); + } + } + else { + CompletableFuture future = tenants.createTenantAsync(tntName, tenantInfo); + future.whenComplete((unused, throwable) -> { + if (logger.isDebugEnabled()) { + logger.debug("Successful async creation of tenant \"{}\"", tntName); + } + }).exceptionally(ex -> { + if (logger.isDebugEnabled()) { + logger.error("Failed async creation of tenant \"{}\"", tntName); + } + return null; + }); + } + } + catch (PulsarAdminException.ConflictException ce) { + if (logger.isDebugEnabled()) { + logger.error("Tenant \"{}\" already exists - skip creation!", tntName); + } + } + catch (PulsarAdminException e) { + throw new PulsarAdapterUnexpectedException( + "Unexpected error when creating pulsar tenant \"" + tntName + "\""); + } + } + // Admin API - delete tenants and namespaces + else { + try { + Namespaces namespaces = pulsarAdmin.namespaces(); + int nsNum = namespaces.getNamespaces(tntName).size(); + + // Only delete a tenant when there is no underlying namespaces + if ( nsNum == 0 ) { + if (!asyncApi) { + tenants.deleteTenant(tntName); + if (logger.isDebugEnabled()) { + logger.debug("Successful sync deletion of tenant \"{}\"", tntName); + } + } + else { + CompletableFuture future = tenants.deleteTenantAsync(tntName); + future.whenComplete((unused, throwable) -> { + if (logger.isDebugEnabled()) { + logger.debug("Successful async deletion of tenant \"{}\"", tntName); + } + }).exceptionally(ex -> { + if (logger.isDebugEnabled()) { + logger.error("Failed async deletion of tenant \"{}\"", tntName); + } + return null; + }); + } + } + } + catch (PulsarAdminException.NotFoundException nfe) { + if (logger.isDebugEnabled()) { + logger.error("Tenant \"{}\" doesn't exists - skip deletion!", tntName); + } + } + catch (PulsarAdminException e) { + throw new PulsarAdapterUnexpectedException( + "Unexpected error when deleting pulsar tenant \"" + tntName + "\""); + } + } + } + + return null; + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/AdminTopicOp.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/AdminTopicOp.java new file mode 100644 index 000000000..507c13199 --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/AdminTopicOp.java @@ -0,0 +1,184 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.ops; + +import io.nosqlbench.adapter.pulsar.exception.PulsarAdapterUnexpectedException; +import io.nosqlbench.adapter.pulsar.util.PulsarAdapterMetrics; +import org.apache.commons.lang3.StringUtils; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; +import org.apache.pulsar.client.admin.PulsarAdmin; +import org.apache.pulsar.client.admin.PulsarAdminException; +import org.apache.pulsar.client.admin.Topics; + +import java.util.Set; +import java.util.concurrent.CompletableFuture; + +public class AdminTopicOp extends PulsarAdminOp { + + private final static Logger logger = LogManager.getLogger(AdminTopicOp.class); + + private final String topicName; + private final boolean enablePart; + private final int partNum; + + public AdminTopicOp(PulsarAdapterMetrics pulsarAdapterMetrics, + PulsarAdmin pulsarAdmin, + boolean asyncApi, + boolean adminDelOp, + String topicName, + boolean enablePart, + int partNum) { + super(pulsarAdapterMetrics, pulsarAdmin, asyncApi, adminDelOp); + + this.topicName = topicName; + this.enablePart = enablePart; + this.partNum = partNum; + } + + @Override + public Void apply(long value) { + + // Do nothing if the topic name is empty + if ( !StringUtils.isBlank(topicName) ) { + Topics topics = pulsarAdmin.topics(); + + try { + // Create the topic + if (!adminDelOp) { + if (!enablePart) { + if (!asyncApi) { + topics.createNonPartitionedTopic(topicName); + if (logger.isDebugEnabled()) { + logger.debug("Successful sync creation of non-partitioned topic \"{}\"", topicName); + } + } else { + CompletableFuture future = topics.createNonPartitionedTopicAsync(topicName); + future.whenComplete((unused, throwable) -> { + if (logger.isDebugEnabled()) { + logger.debug("Successful async creation of non-partitioned topic \"{}\"", topicName); + } + }).exceptionally(ex -> { + if (logger.isDebugEnabled()) { + logger.error("Failed async creation non-partitioned topic \"{}\"", topicName); + return null; + } + return null; + }); + } + } else { + if (!asyncApi) { + topics.createPartitionedTopic(topicName, partNum); + if (logger.isDebugEnabled()) { + logger.debug( + "Successful sync creation of partitioned topic \"{} (partition_num: {}\")", + topicName, partNum); + } + } else { + CompletableFuture future = topics.createPartitionedTopicAsync(topicName, partNum); + future.whenComplete((unused, throwable) -> { + if (logger.isDebugEnabled()) { + logger.debug( + "Successful async creation of partitioned topic \"{} (partition_num: {}\")", + topicName, partNum); + } + }) + .exceptionally(ex -> { + if (logger.isDebugEnabled()) { + logger.error( + "Successful async creation of partitioned topic \"{} (partition_num: {}\")", + topicName, partNum); + } + return null; + }); + } + } + } + // Delete the topic + else { + if (!enablePart) { + if (!asyncApi) { + topics.delete(topicName); + if (logger.isDebugEnabled()) { + logger.debug( + "Successful sync deletion of non-partitioned topic \"{}\"", + topicName); + } + } else { + CompletableFuture future = topics.deleteAsync(topicName); + future.whenComplete((unused, throwable) -> { + if (logger.isDebugEnabled()) { + logger.debug( + "Successful async deletion of non-partitioned topic \"{}\"", + topicName); + } + }) + .exceptionally(ex -> { + if (logger.isDebugEnabled()) { + logger.error( + "Failed async deletion of non-partitioned topic \"{}\"", + topicName); + } + return null; + }); + } + } else { + if (!asyncApi) { + topics.deletePartitionedTopic(topicName); + if (logger.isDebugEnabled()) { + logger.debug( + "Successful sync deletion of partitioned topic \"{} (partition_num: {}\")", + topicName, partNum); + } + } else { + CompletableFuture future = topics.deletePartitionedTopicAsync(topicName); + future.whenComplete((unused, throwable) -> { + if (logger.isDebugEnabled()) { + logger.debug( + "Successful async deletion of partitioned topic \"{} (partition_num: {}\")", + topicName, partNum); + } + }).exceptionally(ex -> { + if (logger.isDebugEnabled()) { + logger.error( + "Failed async deletion of partitioned topic \"{} (partition_num: {}\")", + topicName, partNum); + } + return null; + }); + } + } + } + } + catch (PulsarAdminException.NotFoundException nfe) { + if (logger.isDebugEnabled()) { + logger.error("Topic \"{}\" doesn't exists - skip deletion!", topicName); + } + } + catch (PulsarAdminException e) { + String errMsg = String.format("Unexpected error when %s pulsar topic: %s (partition enabled: %b; partition number: %d)", + (!adminDelOp ? "creating" : "deleting"), + topicName, + enablePart, + partNum); + throw new PulsarAdapterUnexpectedException(errMsg); + } + } + + return null; + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/MessageConsumerOp.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/MessageConsumerOp.java new file mode 100644 index 000000000..40b81ceac --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/MessageConsumerOp.java @@ -0,0 +1,300 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.ops; + +import com.codahale.metrics.Timer; +import io.nosqlbench.adapter.pulsar.exception.PulsarAdapterAsyncOperationFailedException; +import io.nosqlbench.adapter.pulsar.exception.PulsarAdapterUnexpectedException; +import io.nosqlbench.adapter.pulsar.util.*; +import org.apache.commons.lang3.StringUtils; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; +import org.apache.pulsar.client.api.*; +import org.apache.pulsar.client.api.schema.GenericObject; +import org.apache.pulsar.client.api.schema.GenericRecord; +import org.apache.pulsar.client.api.transaction.Transaction; +import org.apache.pulsar.common.schema.KeyValue; +import org.apache.pulsar.shade.org.apache.avro.AvroRuntimeException; + +import java.util.concurrent.CompletableFuture; +import java.util.concurrent.ExecutionException; +import java.util.concurrent.TimeUnit; +import java.util.concurrent.TimeoutException; +import java.util.function.Function; +import java.util.function.Supplier; + +public class MessageConsumerOp extends PulsarClientOp { + + private final static Logger logger = LogManager.getLogger(MessageConsumerOp.class); + + private final boolean useTransact; + private final boolean seqTracking; + private final Supplier transactSupplier; + private final String payloadRttField; + private final EndToEndStartingTimeSource e2eStartingTimeSrc; + private final Function receivedMessageSequenceTrackerForTopic; + private final Consumer consumer; + private final int consumerTimeoutInSec; + + public MessageConsumerOp(PulsarAdapterMetrics pulsarAdapterMetrics, + PulsarClient pulsarClient, + Schema pulsarSchema, + boolean asyncApi, + boolean useTransact, + boolean seqTracking, + Supplier transactSupplier, + String payloadRttField, + EndToEndStartingTimeSource e2eStartingTimeSrc, + Function receivedMessageSequenceTrackerForTopic, + Consumer consumer, + int consumerTimeoutInSec) { + super(pulsarAdapterMetrics, pulsarClient, pulsarSchema, asyncApi); + + this.useTransact = useTransact; + this.seqTracking = seqTracking; + this.transactSupplier = transactSupplier; + this.payloadRttField = payloadRttField; + this.e2eStartingTimeSrc = e2eStartingTimeSrc; + this.receivedMessageSequenceTrackerForTopic = receivedMessageSequenceTrackerForTopic; + this.consumer = consumer; + this.consumerTimeoutInSec = consumerTimeoutInSec; + } + + @Override + public Object apply(long value) { + final Transaction transaction; + if (useTransact) { + // if you are in a transaction you cannot set the schema per-message + transaction = transactSupplier.get(); + } + else { + transaction = null; + } + + if (!asyncApi) { + try { + Message message; + + if (consumerTimeoutInSec <= 0) { + // wait forever + message = consumer.receive(); + } + else { + message = consumer.receive(consumerTimeoutInSec, TimeUnit.SECONDS); + if (message == null) { + if ( logger.isDebugEnabled() ) { + logger.debug("Failed to sync-receive a message before time out ({} seconds)", consumerTimeoutInSec); + } + } + } + + handleMessage(transaction, message); + } + catch (Exception e) { + throw new PulsarAdapterUnexpectedException("" + + "Sync message receiving failed - timeout value: " + consumerTimeoutInSec + " seconds "); + } + } + else { + try { + CompletableFuture> msgRecvFuture = consumer.receiveAsync(); + if (useTransact) { + // add commit step + msgRecvFuture = msgRecvFuture.thenCompose(msg -> { + Timer.Context ctx = transactionCommitTimer.time(); + return transaction + .commit() + .whenComplete((m,e) -> ctx.close()) + .thenApply(v-> msg); + } + ); + } + + msgRecvFuture.thenAccept(message -> { + try { + handleMessage(transaction, message); + } catch (PulsarClientException | TimeoutException e) { + throw new PulsarAdapterAsyncOperationFailedException(e); + } catch (InterruptedException e) { + Thread.currentThread().interrupt(); + } catch (ExecutionException e) { + throw new PulsarAdapterAsyncOperationFailedException(e.getCause()); + } + }).exceptionally(ex -> { + throw new PulsarAdapterAsyncOperationFailedException(ex); + }); + } + catch (Exception e) { + throw new PulsarAdapterUnexpectedException(e); + } + } + + return null; + } + + private void handleMessage(Transaction transaction, Message message) + throws PulsarClientException, InterruptedException, ExecutionException, TimeoutException { + + // acknowledge the message as soon as possible + if (!useTransact) { + consumer.acknowledgeAsync(message.getMessageId()) + .get(consumerTimeoutInSec, TimeUnit.SECONDS); + } else { + consumer.acknowledgeAsync(message.getMessageId(), transaction) + .get(consumerTimeoutInSec, TimeUnit.SECONDS); + + // little problem: here we are counting the "commit" time + // inside the overall time spent for the execution of the consume operation + // we should refactor this operation as for PulsarProducerOp, and use the passed callback + // to track with precision the time spent for the operation and for the commit + try (Timer.Context ctx = transactionCommitTimer.time()) { + transaction.commit().get(); + } + } + + if (logger.isDebugEnabled()) { + Object decodedPayload = message.getValue(); + if (decodedPayload instanceof GenericObject) { + // GenericObject is a wrapper for Primitives, for AVRO/JSON structs and for KeyValu + // we fall here with a configured AVRO schema or with AUTO_CONSUME + GenericObject object = (GenericObject) decodedPayload; + logger.debug("({}) message received: msg-key={}; msg-properties={}; msg-payload={}", + consumer.getConsumerName(), + message.getKey(), + message.getProperties(), + object.getNativeObject() + ""); + } + else { + logger.debug("({}) message received: msg-key={}; msg-properties={}; msg-payload={}", + consumer.getConsumerName(), + message.getKey(), + message.getProperties(), + new String(message.getData())); + } + } + + if (!payloadRttField.isEmpty()) { + boolean done = false; + Object decodedPayload = message.getValue(); + Long extractedSendTime = null; + // if Pulsar is able to decode this it is better to let it do the work + // because Pulsar caches the Schema, handles Schema evolution + // as much efficiently as possible + if (decodedPayload instanceof GenericRecord) { // AVRO and AUTO_CONSUME + final GenericRecord pulsarGenericRecord = (GenericRecord) decodedPayload; + + Object field = null; + // KeyValue is a special wrapper in Pulsar to represent a pair of values + // a Key and a Value + Object nativeObject = pulsarGenericRecord.getNativeObject(); + if (nativeObject instanceof KeyValue) { + KeyValue keyValue = (KeyValue) nativeObject; + // look into the Key + if (keyValue.getKey() instanceof GenericRecord) { + GenericRecord keyPart = (GenericRecord) keyValue.getKey(); + try { + field = keyPart.getField(payloadRttField); + } catch (AvroRuntimeException err) { + // field is not in the key + logger.error("Cannot find {} in key {}: {}", payloadRttField, keyPart, err + ""); + } + } + // look into the Value + if (keyValue.getValue() instanceof GenericRecord && field == null) { + GenericRecord valuePart = (GenericRecord) keyValue.getValue(); + try { + field = valuePart.getField(payloadRttField); + } catch (AvroRuntimeException err) { + // field is not in the value + logger.error("Cannot find {} in value {}: {}", payloadRttField, valuePart, err + ""); + } + } + if (field == null) { + throw new RuntimeException("Cannot find field {}" + payloadRttField + " in " + keyValue.getKey() + " and " + keyValue.getValue()); + } + } else { + field = pulsarGenericRecord.getField(payloadRttField); + } + + if (field != null) { + if (field instanceof Number) { + extractedSendTime = ((Number) field).longValue(); + } else { + extractedSendTime = Long.valueOf(field.toString()); + } + } else { + logger.error("Cannot find {} in value {}", payloadRttField, pulsarGenericRecord); + } + done = true; + } + if (!done) { + org.apache.avro.Schema avroSchema = getAvroSchemaFromConfiguration(); + org.apache.avro.generic.GenericRecord avroGenericRecord = + PulsarAvroSchemaUtil.GetGenericRecord_ApacheAvro(avroSchema, message.getData()); + if (avroGenericRecord.hasField(payloadRttField)) { + extractedSendTime = (Long) avroGenericRecord.get(payloadRttField); + } + } + if (extractedSendTime != null) { + // fallout expects latencies in "ns" and not in "ms" + long delta = TimeUnit.MILLISECONDS + .toNanos(System.currentTimeMillis() - extractedSendTime); + payloadRttHistogram.update(delta); + } + } + + // keep track end-to-end message processing latency + if (e2eStartingTimeSrc != EndToEndStartingTimeSource.NONE) { + long startTimeStamp = 0L; + + switch (e2eStartingTimeSrc) { + case MESSAGE_PUBLISH_TIME: + startTimeStamp = message.getPublishTime(); + break; + case MESSAGE_EVENT_TIME: + startTimeStamp = message.getEventTime(); + break; + case MESSAGE_PROPERTY_E2E_STARTING_TIME: + String startingTimeProperty = message.getProperty("e2e_starting_time"); + startTimeStamp = startingTimeProperty != null ? Long.parseLong(startingTimeProperty) : 0L; + break; + } + + if (startTimeStamp != 0L) { + long e2eMsgLatency = System.currentTimeMillis() - startTimeStamp; + e2eMsgProcLatencyHistogram.update(e2eMsgLatency); + } + } + + // keep track of message errors and update error counters + if (seqTracking) checkAndUpdateMessageErrorCounter(message); + + int messageSize = message.getData().length; + messageSizeHistogram.update(messageSize); + } + + private void checkAndUpdateMessageErrorCounter(Message message) { + String msgSeqIdStr = message.getProperty(PulsarAdapterUtil.MSG_SEQUENCE_NUMBER); + + if ( !StringUtils.isBlank(msgSeqIdStr) ) { + long sequenceNumber = Long.parseLong(msgSeqIdStr); + ReceivedMessageSequenceTracker receivedMessageSequenceTracker = + receivedMessageSequenceTrackerForTopic.apply(message.getTopicName()); + receivedMessageSequenceTracker.sequenceNumberReceived(sequenceNumber); + } + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/MessageProducerOp.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/MessageProducerOp.java new file mode 100644 index 000000000..39c346473 --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/MessageProducerOp.java @@ -0,0 +1,289 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.ops; + +import com.codahale.metrics.Timer; +import io.nosqlbench.adapter.pulsar.exception.PulsarAdapterAsyncOperationFailedException; +import io.nosqlbench.adapter.pulsar.exception.PulsarAdapterUnexpectedException; +import io.nosqlbench.adapter.pulsar.util.MessageSequenceNumberSendingHandler; +import io.nosqlbench.adapter.pulsar.util.PulsarAdapterMetrics; +import io.nosqlbench.adapter.pulsar.util.PulsarAdapterUtil; +import io.nosqlbench.adapter.pulsar.util.PulsarAvroSchemaUtil; +import org.apache.commons.lang3.StringUtils; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; +import org.apache.pulsar.client.api.*; +import org.apache.pulsar.client.api.schema.GenericRecord; +import org.apache.pulsar.client.api.schema.KeyValueSchema; +import org.apache.pulsar.client.api.transaction.Transaction; +import org.apache.pulsar.client.impl.schema.generic.GenericAvroSchema; +import org.apache.pulsar.common.schema.KeyValue; +import org.apache.pulsar.common.schema.SchemaType; + +import java.nio.charset.StandardCharsets; +import java.util.HashMap; +import java.util.Map; +import java.util.Set; +import java.util.concurrent.CompletableFuture; +import java.util.concurrent.ExecutionException; +import java.util.function.Supplier; + +public class MessageProducerOp extends PulsarClientOp { + + private final static Logger logger = LogManager.getLogger("MessageProducerOp"); + + private final boolean useTransact; + private final boolean seqTracking; + private final Supplier transactSupplier; + private final Set errSimuTypeSet; + private final Producer producer; + private final String msgKey; + private final String msgPropRawJsonStr; + private final String msgValue; + + private final Map msgProperties = new HashMap<>(); + private final ThreadLocal> MessageSequenceNumberSendingHandlersThreadLocal = + ThreadLocal.withInitial(HashMap::new); + + public MessageProducerOp(PulsarAdapterMetrics pulsarAdapterMetrics, + PulsarClient pulsarClient, + Schema pulsarSchema, + boolean asyncApi, + boolean useTransact, + boolean seqTracking, + Supplier transactSupplier, + Set errSimuTypeSet, + Producer producer, + String msgKey, + String msgProp, + String msgValue) { + super(pulsarAdapterMetrics, pulsarClient, pulsarSchema, asyncApi); + + this.useTransact = useTransact; + this.seqTracking = seqTracking; + this.transactSupplier = transactSupplier; + this.errSimuTypeSet = errSimuTypeSet; + this.producer = producer; + this.msgKey = msgKey; + this.msgPropRawJsonStr = msgProp; + this.msgValue = msgValue; + + getMsgPropMapFromRawJsonStr(); + } + + private MessageSequenceNumberSendingHandler getMessageSequenceNumberSendingHandler(String topicName) { + return MessageSequenceNumberSendingHandlersThreadLocal.get() + .computeIfAbsent(topicName, k -> new MessageSequenceNumberSendingHandler()); + } + + // Check if msgPropJonStr is valid JSON string with a collection of key/value pairs + // - if Yes, convert it to a map + // - otherwise, log an error message and ignore message properties without throwing a runtime exception + private void getMsgPropMapFromRawJsonStr() { + if (!StringUtils.isBlank(msgPropRawJsonStr)) { + try { + msgProperties.putAll(PulsarAdapterUtil.convertJsonToMap(msgPropRawJsonStr)); + } + catch (Exception e) { + logger.error( + "Error parsing message property JSON string {}, ignore message properties!", + msgPropRawJsonStr); + } + } + + if (seqTracking) { + long nextSequenceNumber = getMessageSequenceNumberSendingHandler(producer.getTopic()) + .getNextSequenceNumber(errSimuTypeSet); + msgProperties.put(PulsarAdapterUtil.MSG_SEQUENCE_NUMBER, String.valueOf(nextSequenceNumber)); + } + } + + @Override + public Object apply(long value) { + + TypedMessageBuilder typedMessageBuilder; + + final Transaction transaction; + if (useTransact) { + // if you are in a transaction you cannot set the schema per-message + transaction = transactSupplier.get(); + typedMessageBuilder = producer.newMessage(transaction); + } + else { + transaction = null; + typedMessageBuilder = producer.newMessage(pulsarSchema); + } + + // set message key + if ( !StringUtils.isBlank(msgKey) && !(pulsarSchema instanceof KeyValueSchema) ) { + typedMessageBuilder = typedMessageBuilder.key(msgKey); + } + + // set message properties + if ( !msgPropRawJsonStr.isEmpty() ) { + typedMessageBuilder = typedMessageBuilder.properties(msgProperties); + } + + // set message payload + int messageSize; + SchemaType schemaType = pulsarSchema.getSchemaInfo().getType(); + if (pulsarSchema instanceof KeyValueSchema) { + +// // {KEY IN JSON}||{VALUE IN JSON} +// int separator = msgValue.indexOf("}||{"); +// if (separator < 0) { +// throw new IllegalArgumentException("KeyValue payload MUST be in form {KEY IN JSON}||{VALUE IN JSON} (with 2 pipes that separate the KEY part from the VALUE part)"); +// } +// String keyInput = msgValue.substring(0, separator + 1); +// String valueInput = msgValue.substring(separator + 3); + + KeyValueSchema keyValueSchema = (KeyValueSchema) pulsarSchema; + org.apache.avro.Schema avroSchema = getAvroSchemaFromConfiguration(); + GenericRecord payload = PulsarAvroSchemaUtil.GetGenericRecord_PulsarAvro( + (GenericAvroSchema) keyValueSchema.getValueSchema(), + avroSchema, + msgValue + ); + + org.apache.avro.Schema avroSchemaForKey = getKeyAvroSchemaFromConfiguration(); + GenericRecord key = PulsarAvroSchemaUtil.GetGenericRecord_PulsarAvro( + (GenericAvroSchema) keyValueSchema.getKeySchema(), + avroSchemaForKey, + msgKey + ); + + typedMessageBuilder = typedMessageBuilder.value(new KeyValue(key, payload)); + // TODO: add a way to calculate the message size for KEY_VALUE messages + messageSize = msgKey.length() + msgValue.length(); + } + else if (PulsarAdapterUtil.isAvroSchemaTypeStr(schemaType.name())) { + GenericRecord payload = PulsarAvroSchemaUtil.GetGenericRecord_PulsarAvro( + (GenericAvroSchema) pulsarSchema, + pulsarSchema.getSchemaInfo().getSchemaDefinition(), + msgValue + ); + typedMessageBuilder = typedMessageBuilder.value(payload); + // TODO: add a way to calculate the message size for AVRO messages + messageSize = msgValue.length(); + } else { + byte[] array = msgValue.getBytes(StandardCharsets.UTF_8); + typedMessageBuilder = typedMessageBuilder.value(array); + messageSize = array.length; + } + + messageSizeHistogram.update(messageSize); + + //TODO: add error handling with failed message production + if (!asyncApi) { + try { + logger.trace("Sending message"); + typedMessageBuilder.send(); + + if (useTransact) { + try (Timer.Context ctx = transactionCommitTimer.time()) { + transaction.commit().get(); + } + } + + if (logger.isDebugEnabled()) { + if (PulsarAdapterUtil.isAvroSchemaTypeStr(schemaType.name())) { + org.apache.avro.Schema avroSchema = getAvroSchemaFromConfiguration(); + org.apache.avro.generic.GenericRecord avroGenericRecord = + PulsarAvroSchemaUtil.GetGenericRecord_ApacheAvro(avroSchema, msgValue); + + logger.debug("({}) Sync message sent: msg-key={}; msg-properties={}; msg-payload={})", + producer.getProducerName(), + msgKey, + msgProperties, + avroGenericRecord.toString()); + } + else { + logger.debug("({}) Sync message sent; msg-key={}; msg-properties={}; msg-payload={}", + producer.getProducerName(), + msgKey, + msgProperties, + msgValue); + } + } + } + catch (PulsarClientException | ExecutionException | InterruptedException pce) { + String errMsg = + "Sync message sending failed: " + + "key - " + msgKey + "; " + + "properties - " + msgProperties + "; " + + "payload - " + msgValue; + + logger.trace(errMsg); + + throw new PulsarAdapterUnexpectedException(errMsg); + } + } + else { + try { + // we rely on blockIfQueueIsFull in order to throttle the request in this case + CompletableFuture future = typedMessageBuilder.sendAsync(); + + if (useTransact) { + // add commit step + future = future.thenCompose(msg -> { + Timer.Context ctx = transactionCommitTimer.time(); + return transaction + .commit() + .whenComplete((m,e) -> ctx.close()) + .thenApply(v-> msg); + } + ); + } + + future.whenComplete((messageId, error) -> { + if (logger.isDebugEnabled()) { + if (PulsarAdapterUtil.isAvroSchemaTypeStr(schemaType.name())) { + org.apache.avro.Schema avroSchema = getAvroSchemaFromConfiguration(); + org.apache.avro.generic.GenericRecord avroGenericRecord = + PulsarAvroSchemaUtil.GetGenericRecord_ApacheAvro(avroSchema, msgValue); + + logger.debug("({}) Aysnc message sent: msg-key={}; msg-properties={}; msg-payload={})", + producer.getProducerName(), + msgKey, + msgProperties, + avroGenericRecord.toString()); + } + else { + logger.debug("({}) Aysnc message sent: msg-key={}; msg-properties={}; msg-payload={}", + producer.getProducerName(), + msgKey, + msgProperties, + msgValue); + } + } + }).exceptionally(ex -> { + logger.error("Async message sending failed: " + + "key - " + msgKey + "; " + + "properties - " + msgProperties + "; " + + "payload - " + msgValue); + + throw new PulsarAdapterAsyncOperationFailedException(ex); + }); + } + catch (Exception e) { + throw new PulsarAdapterUnexpectedException(e); + } + } + + return null; + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/MessageReaderOp.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/MessageReaderOp.java new file mode 100644 index 000000000..18ba4ac3e --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/MessageReaderOp.java @@ -0,0 +1,48 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.ops; + +import io.nosqlbench.adapter.pulsar.util.PulsarAdapterMetrics; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; +import org.apache.pulsar.client.api.PulsarClient; +import org.apache.pulsar.client.api.Reader; +import org.apache.pulsar.client.api.Schema; + +public class MessageReaderOp extends PulsarClientOp { + + private final static Logger logger = LogManager.getLogger(MessageReaderOp.class); + + private final Reader reader; + + public MessageReaderOp(PulsarAdapterMetrics pulsarAdapterMetrics, + PulsarClient pulsarClient, + Schema pulsarSchema, + boolean asyncApi, + Reader reader) { + super(pulsarAdapterMetrics, pulsarClient, pulsarSchema, asyncApi); + + this.reader = reader; + } + + @Override + public Object apply(long value) { + // TODO: implement the Pulsar reader logic when needed + // at the moment, the reader API support from the NB Pulsar driver is disabled + return null; + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/PulsarAdminOp.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/PulsarAdminOp.java new file mode 100644 index 000000000..b1be645d2 --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/PulsarAdminOp.java @@ -0,0 +1,36 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.ops; + +import io.nosqlbench.adapter.pulsar.util.PulsarAdapterMetrics; +import io.nosqlbench.engine.api.activityimpl.uniform.flowtypes.CycleOp; +import org.apache.pulsar.client.admin.PulsarAdmin; + +public abstract class PulsarAdminOp extends PulsarOp { + protected PulsarAdmin pulsarAdmin; + protected boolean adminDelOp; + + public PulsarAdminOp(PulsarAdapterMetrics pulsarAdapterMetrics, + PulsarAdmin pulsarAdmin, + boolean asyncApi, + boolean adminDelOp) { + super(pulsarAdapterMetrics, asyncApi); + + this.pulsarAdmin = pulsarAdmin; + this.adminDelOp = adminDelOp; + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/PulsarClientOp.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/PulsarClientOp.java new file mode 100644 index 000000000..15ea44b15 --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/PulsarClientOp.java @@ -0,0 +1,89 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.ops; + +import com.codahale.metrics.Histogram; +import com.codahale.metrics.Timer; +import io.nosqlbench.adapter.pulsar.util.PulsarAdapterMetrics; +import io.nosqlbench.adapter.pulsar.util.PulsarAvroSchemaUtil; +import org.apache.pulsar.client.api.PulsarClient; +import org.apache.pulsar.client.api.Schema; +import org.apache.pulsar.client.api.schema.KeyValueSchema; +import org.apache.pulsar.common.schema.SchemaType; + +public abstract class PulsarClientOp extends PulsarOp { + protected final PulsarClient pulsarClient; + protected final Schema pulsarSchema; + + // Pulsar KeyValue schema + private org.apache.avro.Schema avroSchema; + private org.apache.avro.Schema avroKeySchema; + + protected final Histogram messageSizeHistogram; + protected final Histogram payloadRttHistogram; + protected final Histogram e2eMsgProcLatencyHistogram; + + protected final Timer transactionCommitTimer; + + public PulsarClientOp(PulsarAdapterMetrics pulsarAdapterMetrics, + PulsarClient pulsarClient, + Schema pulsarScheam, + boolean asyncApi) { + super (pulsarAdapterMetrics, asyncApi); + + this.pulsarClient = pulsarClient; + this.pulsarSchema = pulsarScheam; + + this.messageSizeHistogram = pulsarAdapterMetrics.getMessageSizeHistogram(); + this.payloadRttHistogram = pulsarAdapterMetrics.getPayloadRttHistogram(); + this.e2eMsgProcLatencyHistogram = pulsarAdapterMetrics.getE2eMsgProcLatencyHistogram(); + this.transactionCommitTimer = pulsarAdapterMetrics.getCommitTransactionTimer(); + } + + protected org.apache.avro.Schema getAvroSchemaFromConfiguration() { + // no need for synchronization, this is only a cache + // in case of the race we will parse the string twice, not a big + if (avroSchema == null) { + if (pulsarSchema.getSchemaInfo().getType() == SchemaType.KEY_VALUE) { + KeyValueSchema kvSchema = (KeyValueSchema) pulsarSchema; + Schema valueSchema = kvSchema.getValueSchema(); + String avroDefStr = valueSchema.getSchemaInfo().getSchemaDefinition(); + avroSchema = PulsarAvroSchemaUtil.GetSchema_ApacheAvro(avroDefStr); + } else { + String avroDefStr = pulsarSchema.getSchemaInfo().getSchemaDefinition(); + avroSchema = PulsarAvroSchemaUtil.GetSchema_ApacheAvro(avroDefStr); + } + } + return avroSchema; + } + + protected org.apache.avro.Schema getKeyAvroSchemaFromConfiguration() { + // no need for synchronization, this is only a cache + // in case of the race we will parse the string twice, not a big + if (avroKeySchema == null) { + if (pulsarSchema.getSchemaInfo().getType() == SchemaType.KEY_VALUE) { + KeyValueSchema kvSchema = (KeyValueSchema) pulsarSchema; + Schema keySchema = kvSchema.getKeySchema(); + String avroDefStr = keySchema.getSchemaInfo().getSchemaDefinition(); + avroKeySchema = PulsarAvroSchemaUtil.GetSchema_ApacheAvro(avroDefStr); + } else { + throw new RuntimeException("We are not using KEY_VALUE schema, so no Schema for the Key!"); + } + } + return avroKeySchema; + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/PulsarOp.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/PulsarOp.java new file mode 100644 index 000000000..a3e0b87b1 --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/ops/PulsarOp.java @@ -0,0 +1,31 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.ops; + +import io.nosqlbench.adapter.pulsar.util.PulsarAdapterMetrics; +import io.nosqlbench.engine.api.activityimpl.uniform.flowtypes.CycleOp; + +public abstract class PulsarOp implements CycleOp { + protected final boolean asyncApi; + protected final PulsarAdapterMetrics pulsarAdapterMetrics; + + public PulsarOp(PulsarAdapterMetrics pulsarAdapterMetrics, boolean asyncApi) { + this.pulsarAdapterMetrics = pulsarAdapterMetrics; + this.asyncApi = asyncApi; + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/EndToEndStartingTimeSource.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/EndToEndStartingTimeSource.java new file mode 100644 index 000000000..7a9f39680 --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/EndToEndStartingTimeSource.java @@ -0,0 +1,26 @@ +package io.nosqlbench.adapter.pulsar.util; + +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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. + */ + + +public enum EndToEndStartingTimeSource { + NONE, // no end-to-end latency calculation + MESSAGE_PUBLISH_TIME, // use message publish timestamp + MESSAGE_EVENT_TIME, // use message event timestamp + MESSAGE_PROPERTY_E2E_STARTING_TIME // use message property called "e2e_starting_time" as the timestamp +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/MessageSequenceNumberSendingHandler.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/MessageSequenceNumberSendingHandler.java new file mode 100644 index 000000000..9ff2aad5f --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/MessageSequenceNumberSendingHandler.java @@ -0,0 +1,109 @@ +package io.nosqlbench.adapter.pulsar.util; + +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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. + */ + + +import io.nosqlbench.adapter.pulsar.util.PulsarAdapterUtil; +import org.apache.commons.lang3.RandomUtils; + +import java.util.ArrayDeque; +import java.util.Arrays; +import java.util.Queue; +import java.util.Set; + +/** + * Handles adding a monotonic sequence number to message properties of sent messages + */ +public class MessageSequenceNumberSendingHandler { + static final int SIMULATED_ERROR_PROBABILITY_PERCENTAGE = 10; + long number = 1; + Queue outOfOrderNumbers; + + public long getNextSequenceNumber(Set simulatedErrorTypes) { + return getNextSequenceNumber(simulatedErrorTypes, SIMULATED_ERROR_PROBABILITY_PERCENTAGE); + } + + long getNextSequenceNumber(Set simulatedErrorTypes, int errorProbabilityPercentage) { + simulateError(simulatedErrorTypes, errorProbabilityPercentage); + return nextNumber(); + } + + private void simulateError(Set simulatedErrorTypes, int errorProbabilityPercentage) { + if (!simulatedErrorTypes.isEmpty() && shouldSimulateError(errorProbabilityPercentage)) { + int selectIndex = 0; + int numberOfErrorTypes = simulatedErrorTypes.size(); + if (numberOfErrorTypes > 1) { + // pick one of the simulated error type randomly + selectIndex = RandomUtils.nextInt(0, numberOfErrorTypes); + } + PulsarAdapterUtil.SEQ_ERROR_SIMU_TYPE errorType = simulatedErrorTypes.stream() + .skip(selectIndex) + .findFirst() + .get(); + switch (errorType) { + case OutOfOrder: + // simulate message out of order + injectMessagesOutOfOrder(); + break; + case MsgDup: + // simulate message duplication + injectMessageDuplication(); + break; + case MsgLoss: + // simulate message loss + injectMessageLoss(); + break; + } + } + } + + private boolean shouldSimulateError(int errorProbabilityPercentage) { + // Simulate error with the specified probability + return RandomUtils.nextInt(0, 100) < errorProbabilityPercentage; + } + + long nextNumber() { + if (outOfOrderNumbers != null) { + long nextNumber = outOfOrderNumbers.poll(); + if (outOfOrderNumbers.isEmpty()) { + outOfOrderNumbers = null; + } + return nextNumber; + } + return number++; + } + + void injectMessagesOutOfOrder() { + if (outOfOrderNumbers == null) { + outOfOrderNumbers = new ArrayDeque<>(Arrays.asList(number + 2, number, number + 1)); + number += 3; + } + } + + void injectMessageDuplication() { + if (outOfOrderNumbers == null) { + number--; + } + } + + void injectMessageLoss() { + if (outOfOrderNumbers == null) { + number++; + } + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/PulsarAdapterMetrics.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/PulsarAdapterMetrics.java new file mode 100644 index 000000000..ae48803f4 --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/PulsarAdapterMetrics.java @@ -0,0 +1,231 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.util; + +import com.codahale.metrics.Counter; +import com.codahale.metrics.Gauge; +import com.codahale.metrics.Histogram; +import com.codahale.metrics.Timer; +import io.nosqlbench.adapter.pulsar.dispensers.PulsarBaseOpDispenser; +import io.nosqlbench.api.engine.metrics.ActivityMetrics; +import org.apache.commons.lang3.StringUtils; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; +import org.apache.pulsar.client.api.Consumer; +import org.apache.pulsar.client.api.ConsumerStats; +import org.apache.pulsar.client.api.Producer; +import org.apache.pulsar.client.api.ProducerStats; + +import java.util.function.Function; + +public class PulsarAdapterMetrics { + + private final static Logger logger = LogManager.getLogger("PulsarAdapterMetrics"); + + private final PulsarBaseOpDispenser pulsarBaseOpDispenser; + private final String defaultAdapterMetricsPrefix; + + /** + * Pulsar adapter specific metrics + */ + // - message out of sequence error counter + private Counter msgErrOutOfSeqCounter; + // - message loss counter + private Counter msgErrLossCounter; + // - message duplicate (when dedup is enabled) error counter + private Counter msgErrDuplicateCounter; + + private Histogram messageSizeHistogram; + // end-to-end latency + private Histogram e2eMsgProcLatencyHistogram; + // A histogram that tracks payload round-trip-time, based on a user-defined field in some sender + // system which can be interpreted as millisecond epoch time in the system's local time zone. + // This is paired with a field name of the same type to be extracted and reported in a metric + // named 'payload-rtt'. + private Histogram payloadRttHistogram; + + private Timer bindTimer; + private Timer executeTimer; + private Timer createTransactionTimer; + private Timer commitTransactionTimer; + + public PulsarAdapterMetrics(PulsarBaseOpDispenser pulsarBaseOpDispenser, String defaultMetricsPrefix) { + this.pulsarBaseOpDispenser = pulsarBaseOpDispenser; + this.defaultAdapterMetricsPrefix = defaultMetricsPrefix; + } + + public void initPulsarAdapterInstrumentation() { + // Counter metrics + this.msgErrOutOfSeqCounter = + ActivityMetrics.counter( + pulsarBaseOpDispenser, + defaultAdapterMetricsPrefix + "err_msg_oos"); + this.msgErrLossCounter = + ActivityMetrics.counter( + pulsarBaseOpDispenser, + defaultAdapterMetricsPrefix + "err_msg_loss"); + this.msgErrDuplicateCounter = + ActivityMetrics.counter( + pulsarBaseOpDispenser, + defaultAdapterMetricsPrefix + "err_msg_dup"); + + // Histogram metrics + this.messageSizeHistogram = + ActivityMetrics.histogram( + pulsarBaseOpDispenser, + defaultAdapterMetricsPrefix + "message_size", + ActivityMetrics.DEFAULT_HDRDIGITS); + this.e2eMsgProcLatencyHistogram = + ActivityMetrics.histogram( + pulsarBaseOpDispenser, + defaultAdapterMetricsPrefix + "e2e_msg_latency", + ActivityMetrics.DEFAULT_HDRDIGITS); + this.payloadRttHistogram = + ActivityMetrics.histogram( + pulsarBaseOpDispenser, + defaultAdapterMetricsPrefix + "payload_rtt", + ActivityMetrics.DEFAULT_HDRDIGITS); + + // Timer metrics + this.bindTimer = + ActivityMetrics.timer( + pulsarBaseOpDispenser, + defaultAdapterMetricsPrefix + "bind", + ActivityMetrics.DEFAULT_HDRDIGITS); + this.executeTimer = + ActivityMetrics.timer( + pulsarBaseOpDispenser, + defaultAdapterMetricsPrefix + "execute", + ActivityMetrics.DEFAULT_HDRDIGITS); + this.createTransactionTimer = + ActivityMetrics.timer( + pulsarBaseOpDispenser, + defaultAdapterMetricsPrefix + "create_transaction", + ActivityMetrics.DEFAULT_HDRDIGITS); + this.commitTransactionTimer = + ActivityMetrics.timer( + pulsarBaseOpDispenser, + defaultAdapterMetricsPrefix + "commit_transaction", + ActivityMetrics.DEFAULT_HDRDIGITS); + } + + public Counter getMsgErrOutOfSeqCounter() { return this.msgErrOutOfSeqCounter; } + public Counter getMsgErrLossCounter() { return this.msgErrLossCounter; } + public Counter getMsgErrDuplicateCounter() { return this.msgErrDuplicateCounter; } + public Histogram getMessageSizeHistogram() { return this.messageSizeHistogram; } + public Histogram getE2eMsgProcLatencyHistogram() { return this.e2eMsgProcLatencyHistogram; } + public Histogram getPayloadRttHistogram() { return payloadRttHistogram; } + public Timer getBindTimer() { return bindTimer; } + public Timer getExecuteTimer() { return executeTimer; } + public Timer getCreateTransactionTimer() { return createTransactionTimer; } + public Timer getCommitTransactionTimer() { return commitTransactionTimer; } + + + ////////////////////////////////////// + // Pulsar client producer API metrics + ////////////////////////////////////// + // + private static class ProducerGaugeImpl implements Gauge { + private final Producer producer; + private final Function valueExtractor; + + ProducerGaugeImpl(Producer producer, Function valueExtractor) { + this.producer = producer; + this.valueExtractor = valueExtractor; + } + + @Override + public Object getValue() { + // see Pulsar bug https://github.com/apache/pulsar/issues/10100 + // we need to synchronize on producer otherwise we could receive corrupted data + synchronized(producer) { + return valueExtractor.apply(producer.getStats()); + } + } + } + private static Gauge producerSafeExtractMetric(Producer producer, Function valueExtractor) { + return new ProducerGaugeImpl(producer, valueExtractor); + } + + public void registerProducerApiMetrics(Producer producer, String pulsarApiMetricsPrefix) { + String metricsPrefix = defaultAdapterMetricsPrefix; + if (!StringUtils.isBlank(pulsarApiMetricsPrefix)) { + metricsPrefix = pulsarApiMetricsPrefix; + } + + ActivityMetrics.gauge(pulsarBaseOpDispenser, metricsPrefix + "total_bytes_sent", + producerSafeExtractMetric(producer, (s -> s.getTotalBytesSent() + s.getNumBytesSent()))); + ActivityMetrics.gauge(pulsarBaseOpDispenser, metricsPrefix + "total_msg_sent", + producerSafeExtractMetric(producer, (s -> s.getTotalMsgsSent() + s.getNumMsgsSent()))); + ActivityMetrics.gauge(pulsarBaseOpDispenser, metricsPrefix + "total_send_failed", + producerSafeExtractMetric(producer, (s -> s.getTotalSendFailed() + s.getNumSendFailed()))); + ActivityMetrics.gauge(pulsarBaseOpDispenser, metricsPrefix + "total_ack_received", + producerSafeExtractMetric(producer,(s -> s.getTotalAcksReceived() + s.getNumAcksReceived()))); + ActivityMetrics.gauge(pulsarBaseOpDispenser, metricsPrefix + "send_bytes_rate", + producerSafeExtractMetric(producer, ProducerStats::getSendBytesRate)); + ActivityMetrics.gauge(pulsarBaseOpDispenser, metricsPrefix + "send_msg_rate", + producerSafeExtractMetric(producer, ProducerStats::getSendMsgsRate)); + } + + + ////////////////////////////////////// + // Pulsar client consumer API metrics + ////////////////////////////////////// + // + private static class ConsumerGaugeImpl implements Gauge { + private final Consumer consumer; + private final Function valueExtractor; + + ConsumerGaugeImpl(Consumer consumer, Function valueExtractor) { + this.consumer = consumer; + this.valueExtractor = valueExtractor; + } + + @Override + public Object getValue() { + // see Pulsar bug https://github.com/apache/pulsar/issues/10100 + // - this is a bug report for producer stats. + // - assume this also applies to consumer stats. + synchronized(consumer) { + return valueExtractor.apply(consumer.getStats()); + } + } + } + static Gauge consumerSafeExtractMetric(Consumer consumer, Function valueExtractor) { + return new ConsumerGaugeImpl(consumer, valueExtractor); + } + + public void registerConsumerApiMetrics(Consumer consumer, String pulsarApiMetricsPrefix) { + String metricsPrefix = defaultAdapterMetricsPrefix; + if (!StringUtils.isBlank(pulsarApiMetricsPrefix)) { + metricsPrefix = pulsarApiMetricsPrefix; + } + + ActivityMetrics.gauge(pulsarBaseOpDispenser, metricsPrefix + "total_bytes_recv", + consumerSafeExtractMetric(consumer, (s -> s.getTotalBytesReceived() + s.getNumBytesReceived()))); + ActivityMetrics.gauge(pulsarBaseOpDispenser, metricsPrefix + "total_msg_recv", + consumerSafeExtractMetric(consumer, (s -> s.getTotalMsgsReceived() + s.getNumMsgsReceived()))); + ActivityMetrics.gauge(pulsarBaseOpDispenser, metricsPrefix + "total_recv_failed", + consumerSafeExtractMetric(consumer, (s -> s.getTotalReceivedFailed() + s.getNumReceiveFailed()))); + ActivityMetrics.gauge(pulsarBaseOpDispenser, metricsPrefix + "total_acks_sent", + consumerSafeExtractMetric(consumer,(s -> s.getTotalAcksSent() + s.getNumAcksSent()))); + ActivityMetrics.gauge(pulsarBaseOpDispenser, metricsPrefix + "recv_bytes_rate", + consumerSafeExtractMetric(consumer, ConsumerStats::getRateBytesReceived)); + ActivityMetrics.gauge(pulsarBaseOpDispenser, metricsPrefix + "recv_msg_rate", + consumerSafeExtractMetric(consumer, ConsumerStats::getRateMsgsReceived)); + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/PulsarAdapterUtil.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/PulsarAdapterUtil.java new file mode 100644 index 000000000..2150c84bf --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/PulsarAdapterUtil.java @@ -0,0 +1,520 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.util; + +import com.fasterxml.jackson.databind.ObjectMapper; +import io.nosqlbench.adapter.pulsar.exception.PulsarAdapterInvalidParamException; +import io.nosqlbench.adapter.pulsar.exception.PulsarAdapterUnexpectedException; +import org.apache.commons.lang3.StringUtils; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; +import org.apache.pulsar.client.api.Schema; + +import java.io.IOException; +import java.net.URI; +import java.nio.charset.StandardCharsets; +import java.nio.file.Files; +import java.nio.file.Path; +import java.nio.file.Paths; +import java.util.*; +import java.util.stream.Collectors; +import java.util.stream.Stream; + +public class PulsarAdapterUtil { + + private final static Logger logger = LogManager.getLogger(PulsarAdapterUtil.class); + + public static final String MSG_SEQUENCE_NUMBER = "sequence_number"; + + /////// + // Valid document level parameters for Pulsar NB yaml file + public enum DOC_LEVEL_PARAMS { + TOPIC_URI("topic_uri"), + ASYNC_API("async_api"), + USE_TRANSACTION("use_transaction"), + TRANSACT_BATCH_NUM("transact_batch_num"), + ADMIN_DELOP("admin_delop"), + SEQ_TRACKING("seq_tracking"), + RTT_TRACKING_FIELD("payload_traking_field"), + MSG_DEDUP_BROKER("msg_dedup_broker"), + E2E_STARTING_TIME_SOURCE("e2e_starting_time_source"); + + public final String label; + + DOC_LEVEL_PARAMS(String label) { + this.label = label; + } + } + public static boolean isValidDocLevelParam(String param) { + return Arrays.stream(DOC_LEVEL_PARAMS.values()).anyMatch(t -> t.label.equals(param)); + } + + /////// + // Valid Pulsar API type + public enum PULSAR_API_TYPE { + PRODUCER("producer"), + CONSUMER("consumer"), + READER("reader"); + + public final String label; + + PULSAR_API_TYPE(String label) { + this.label = label; + } + } + public static boolean isValidPulsarApiType(String param) { + return Arrays.stream(PULSAR_API_TYPE.values()).anyMatch(t -> t.label.equals(param)); + } + public static String getValidPulsarApiTypeList() { + return Arrays.stream(PULSAR_API_TYPE.values()).map(t -> t.label).collect(Collectors.joining(", ")); + } + + /////// + // Valid persistence type + public enum PERSISTENT_TYPES { + PERSISTENT("persistent"), + NON_PERSISTENT("non-persistent") + ; + + public final String label; + PERSISTENT_TYPES(String label) { + this.label = label; + } + } + public static boolean isValidPersistenceType(String type) { + return Arrays.stream(PERSISTENT_TYPES.values()).anyMatch(t -> t.label.equals(type)); + } + + /////// + // Valid Pulsar client configuration (activity-level settings) + // - https://pulsar.apache.org/docs/en/client-libraries-java/#client + public enum CLNT_CONF_KEY { + serviceUrl("serviceUrl"), + authPulginClassName("authPluginClassName"), + authParams("authParams"), + pperationTimeoutMs("operationTimeoutMs"), + statsIntervalSeconds("statsIntervalSeconds"), + numIoThreads("numIoThreads"), + numListenerThreads("numListenerThreads"), + useTcpNoDelay("useTcpNoDelay"), + enableTls("enableTls"), + tlsTrustCertsFilePath("tlsTrustCertsFilePath"), + tlsAllowInsecureConnection("tlsAllowInsecureConnection"), + tlsHostnameVerificationEnable("tlsHostnameVerificationEnable"), + concurrentLookupRequest("concurrentLookupRequest"), + maxLookupRequest("maxLookupRequest"), + maxNumberOfRejectedRequestPerConnection("maxNumberOfRejectedRequestPerConnection"), + keepAliveIntervalSeconds("keepAliveIntervalSeconds"), + connectionTimeoutMs("connectionTimeoutMs"), + requestTimeoutMs("requestTimeoutMs"), + defaultBackoffIntervalNanos("defaultBackoffIntervalNanos"), + maxBackoffIntervalNanos("maxBackoffIntervalNanos"), + socks5ProxyAddress("socks5ProxyAddress"), + socks5ProxyUsername("socks5ProxyUsername"), + socks5ProxyPassword("socks5ProxyPassword") + ; + + public final String label; + CLNT_CONF_KEY(String label) { + this.label = label; + } + } + public static boolean isValidClientConfItem(String item) { + return Arrays.stream(CLNT_CONF_KEY.values()).anyMatch(t -> t.label.equals(item)); + } + + /////// + // Standard producer configuration (activity-level settings) + // - https://pulsar.apache.org/docs/en/client-libraries-java/#configure-producer + public enum PRODUCER_CONF_STD_KEY { + topicName("topicName"), + producerName("producerName"), + sendTimeoutMs("sendTimeoutMs"), + blockIfQueueFull("blockIfQueueFull"), + maxPendingMessages("maxPendingMessages"), + maxPendingMessagesAcrossPartitions("maxPendingMessagesAcrossPartitions"), + messageRoutingMode("messageRoutingMode"), + hashingScheme("hashingScheme"), + cryptoFailureAction("cryptoFailureAction"), + batchingMaxPublishDelayMicros("batchingMaxPublishDelayMicros"), + batchingMaxMessages("batchingMaxMessages"), + batchingEnabled("batchingEnabled"), + compressionType("compressionType"); + + public final String label; + + PRODUCER_CONF_STD_KEY(String label) { + this.label = label; + } + } + public static boolean isStandardProducerConfItem(String item) { + return Arrays.stream(PRODUCER_CONF_STD_KEY.values()).anyMatch(t -> t.label.equals(item)); + } + + /////// + // Standard consumer configuration (activity-level settings) + // - https://pulsar.apache.org/docs/en/client-libraries-java/#consumer + public enum CONSUMER_CONF_STD_KEY { + topicNames("topicNames"), + topicsPattern("topicsPattern"), + subscriptionName("subscriptionName"), + subscriptionType("subscriptionType"), + receiverQueueSize("receiverQueueSize"), + acknowledgementsGroupTimeMicros("acknowledgementsGroupTimeMicros"), + negativeAckRedeliveryDelayMicros("negativeAckRedeliveryDelayMicros"), + maxTotalReceiverQueueSizeAcrossPartitions("maxTotalReceiverQueueSizeAcrossPartitions"), + consumerName("consumerName"), + ackTimeoutMillis("ackTimeoutMillis"), + tickDurationMillis("tickDurationMillis"), + priorityLevel("priorityLevel"), + cryptoFailureAction("cryptoFailureAction"), + properties("properties"), + readCompacted("readCompacted"), + subscriptionInitialPosition("subscriptionInitialPosition"), + patternAutoDiscoveryPeriod("patternAutoDiscoveryPeriod"), + regexSubscriptionMode("regexSubscriptionMode"), + deadLetterPolicy("deadLetterPolicy"), + autoUpdatePartitions("autoUpdatePartitions"), + replicateSubscriptionState("replicateSubscriptionState"); + + public final String label; + + CONSUMER_CONF_STD_KEY(String label) { + this.label = label; + } + } + public static boolean isStandardConsumerConfItem(String item) { + return Arrays.stream(CONSUMER_CONF_STD_KEY.values()).anyMatch(t -> t.label.equals(item)); + } + + /////// + // Custom consumer configuration (activity-level settings) + // - NOT part of https://pulsar.apache.org/docs/en/client-libraries-java/#consumer + // - NB Pulsar driver consumer operation specific + public enum CONSUMER_CONF_CUSTOM_KEY { + timeout("timeout"); + + public final String label; + + CONSUMER_CONF_CUSTOM_KEY(String label) { + this.label = label; + } + } + public static boolean isCustomConsumerConfItem(String item) { + return Arrays.stream(CONSUMER_CONF_CUSTOM_KEY.values()).anyMatch(t -> t.label.equals(item)); + } + + /////// + // Pulsar subscription type + public enum SUBSCRIPTION_TYPE { + Exclusive("Exclusive"), + Failover("Failover"), + Shared("Shared"), + Key_Shared("Key_Shared"); + + public final String label; + + SUBSCRIPTION_TYPE(String label) { + this.label = label; + } + } + public static boolean isValidSubscriptionType(String item) { + return Arrays.stream(SUBSCRIPTION_TYPE.values()).anyMatch(t -> t.label.equals(item)); + } + public static String getValidSubscriptionTypeList() { + return Arrays.stream(SUBSCRIPTION_TYPE.values()).map(t -> t.label).collect(Collectors.joining(", ")); + } + + /////// + // Standard reader configuration (activity-level settings) + // - https://pulsar.apache.org/docs/en/client-libraries-java/#reader + public enum READER_CONF_STD_KEY { + topicName("topicName"), + receiverQueueSize("receiverQueueSize"), + readerListener("readerListener"), + readerName("readerName"), + subscriptionRolePrefix("subscriptionRolePrefix"), + cryptoKeyReader("cryptoKeyReader"), + cryptoFailureAction("cryptoFailureAction"), + readCompacted("readCompacted"), + resetIncludeHead("resetIncludeHead"); + + public final String label; + + READER_CONF_STD_KEY(String label) { + this.label = label; + } + } + public static boolean isStandardReaderConfItem(String item) { + return Arrays.stream(READER_CONF_STD_KEY.values()).anyMatch(t -> t.label.equals(item)); + } + + /////// + // Custom reader configuration (activity-level settings) + // - NOT part of https://pulsar.apache.org/docs/en/client-libraries-java/#reader + // - NB Pulsar driver reader operation specific + public enum READER_CONF_CUSTOM_KEY { + startMessagePos("startMessagePos"); + + public final String label; + + READER_CONF_CUSTOM_KEY(String label) { + this.label = label; + } + } + public static boolean isCustomReaderConfItem(String item) { + return Arrays.stream(READER_CONF_CUSTOM_KEY.values()).anyMatch(t -> t.label.equals(item)); + } + + /////// + // Valid read positions for a Pulsar reader + public enum READER_MSG_POSITION_TYPE { + earliest("earliest"), + latest("latest"), + custom("custom"); + + public final String label; + + READER_MSG_POSITION_TYPE(String label) { + this.label = label; + } + } + public static boolean isValideReaderStartPosition(String item) { + return Arrays.stream(READER_MSG_POSITION_TYPE.values()).anyMatch(t -> t.label.equals(item)); + } + + /////// + // Pulsar subscription type + public enum SEQ_ERROR_SIMU_TYPE { + OutOfOrder("out_of_order"), + MsgLoss("msg_loss"), + MsgDup("msg_dup"); + + public final String label; + + SEQ_ERROR_SIMU_TYPE(String label) { + this.label = label; + } + + private static final Map MAPPING = new HashMap<>(); + + static { + for (SEQ_ERROR_SIMU_TYPE simuType : values()) { + MAPPING.put(simuType.label, simuType); + MAPPING.put(simuType.label.toLowerCase(), simuType); + MAPPING.put(simuType.label.toUpperCase(), simuType); + MAPPING.put(simuType.name(), simuType); + MAPPING.put(simuType.name().toLowerCase(), simuType); + MAPPING.put(simuType.name().toUpperCase(), simuType); + } + } + + public static Optional parseSimuType(String simuTypeString) { + return Optional.ofNullable(MAPPING.get(simuTypeString.trim())); + } + } + public static boolean isValidSeqErrSimuType(String item) { + return Arrays.stream(SEQ_ERROR_SIMU_TYPE.values()).anyMatch(t -> t.label.equals(item)); + } + public static String getValidSeqErrSimuTypeList() { + return Arrays.stream(SEQ_ERROR_SIMU_TYPE.values()).map(t -> t.label).collect(Collectors.joining(", ")); + } + + /////// + // Valid websocket-producer configuration (activity-level settings) + // TODO: to be added + public enum WEBSKT_PRODUCER_CONF_KEY { + ; + + public final String label; + + WEBSKT_PRODUCER_CONF_KEY(String label) { + this.label = label; + } + } + + /////// + // Valid managed-ledger configuration (activity-level settings) + // TODO: to be added + public enum MANAGED_LEDGER_CONF_KEY { + ; + + public final String label; + MANAGED_LEDGER_CONF_KEY(String label) { + this.label = label; + } + } + + /////// + // Primitive Schema type + public static boolean isPrimitiveSchemaTypeStr(String typeStr) { + boolean isPrimitive = false; + + // Use "BYTES" as the default type if the type string is not explicitly specified + if (StringUtils.isBlank(typeStr)) { + typeStr = "BYTES"; + } + + if (typeStr.equalsIgnoreCase("BOOLEAN") || typeStr.equalsIgnoreCase("INT8") || + typeStr.equalsIgnoreCase("INT16") || typeStr.equalsIgnoreCase("INT32") || + typeStr.equalsIgnoreCase("INT64") || typeStr.equalsIgnoreCase("FLOAT") || + typeStr.equalsIgnoreCase("DOUBLE") || typeStr.equalsIgnoreCase("BYTES") || + typeStr.equalsIgnoreCase("DATE") || typeStr.equalsIgnoreCase("TIME") || + typeStr.equalsIgnoreCase("TIMESTAMP") || typeStr.equalsIgnoreCase("INSTANT") || + typeStr.equalsIgnoreCase("LOCAL_DATE") || typeStr.equalsIgnoreCase("LOCAL_TIME") || + typeStr.equalsIgnoreCase("LOCAL_DATE_TIME")) { + isPrimitive = true; + } + + return isPrimitive; + } + public static Schema getPrimitiveTypeSchema(String typeStr) { + Schema schema; + + switch (typeStr.toUpperCase()) { + case "BOOLEAN": + schema = Schema.BOOL; + break; + case "INT8": + schema = Schema.INT8; + break; + case "INT16": + schema = Schema.INT16; + break; + case "INT32": + schema = Schema.INT32; + break; + case "INT64": + schema = Schema.INT64; + break; + case "FLOAT": + schema = Schema.FLOAT; + break; + case "DOUBLE": + schema = Schema.DOUBLE; + break; + case "DATE": + schema = Schema.DATE; + break; + case "TIME": + schema = Schema.TIME; + break; + case "TIMESTAMP": + schema = Schema.TIMESTAMP; + break; + case "INSTANT": + schema = Schema.INSTANT; + break; + case "LOCAL_DATE": + schema = Schema.LOCAL_DATE; + break; + case "LOCAL_TIME": + schema = Schema.LOCAL_TIME; + break; + case "LOCAL_DATE_TIME": + schema = Schema.LOCAL_DATE_TIME; + break; + // Use BYTES as the default schema type if the type string is not specified + case "": + case "BYTES": + schema = Schema.BYTES; + break; + // Report an error if non-valid, non-empty schema type string is provided + default: + throw new RuntimeException("Invalid Pulsar primitive schema type string : " + typeStr); + } + + return schema; + } + + /////// + // Complex strut type: Avro or Json + public static boolean isAvroSchemaTypeStr(String typeStr) { + return typeStr.equalsIgnoreCase("AVRO"); + } + public static boolean isKeyValueTypeStr(String typeStr) { + return typeStr.equalsIgnoreCase("KEY_VALUE"); + } + + // automatic decode the type from the Registry + public static boolean isAutoConsumeSchemaTypeStr(String typeStr) { + return typeStr.equalsIgnoreCase("AUTO_CONSUME"); + } + public static Schema getAvroSchema(String typeStr, String definitionStr) { + String schemaDefinitionStr = definitionStr; + String filePrefix = "file://"; + Schema schema; + + // Check if payloadStr points to a file (e.g. "file:///path/to/a/file") + if (isAvroSchemaTypeStr(typeStr)) { + if (StringUtils.isBlank(schemaDefinitionStr)) { + throw new PulsarAdapterInvalidParamException( + "Schema definition must be provided for \"Avro\" schema type!"); + } + else if (schemaDefinitionStr.startsWith(filePrefix)) { + try { + Path filePath = Paths.get(URI.create(schemaDefinitionStr)); + schemaDefinitionStr = Files.readString(filePath, StandardCharsets.US_ASCII); + } + catch (IOException ioe) { + throw new PulsarAdapterUnexpectedException( + "Error reading the specified \"Avro\" schema definition file: " + definitionStr + ": " + ioe.getMessage()); + } + } + + schema = PulsarAvroSchemaUtil.GetSchema_PulsarAvro("NBAvro", schemaDefinitionStr); + } + else { + throw new PulsarAdapterInvalidParamException( + "Trying to create a \"Avro\" schema for a non-Avro schema type string: " + typeStr); + } + + return schema; + } + + /////// + // Generate effective key string + public static String buildCacheKey(String... keyParts) { + // Ignore blank keyPart + String joinedKeyStr = + Stream.of(keyParts) + .filter(s -> !StringUtils.isBlank(s)) + .collect(Collectors.joining(",")); + + return Base64.getEncoder().encodeToString(joinedKeyStr.getBytes()); + } + + /////// + // Convert JSON string to a key/value map + public static Map convertJsonToMap(String jsonStr) throws Exception { + ObjectMapper mapper = new ObjectMapper(); + return mapper.readValue(jsonStr, Map.class); + } + + /////// + // Get full namespace name (/) from a Pulsar topic URI + public static String getFullNamespaceName(String topicUri) { + // Get tenant/namespace string + // - topicUri : persistent://// + // - tmpStr : // + // - fullNsName : / + + String tmpStr = StringUtils.substringAfter(topicUri,"://"); + return StringUtils.substringBeforeLast(tmpStr, "/"); + } +} + diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/PulsarAvroSchemaUtil.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/PulsarAvroSchemaUtil.java new file mode 100644 index 000000000..a60b1f0ff --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/PulsarAvroSchemaUtil.java @@ -0,0 +1,123 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.util; + +import org.apache.avro.io.DecoderFactory; +import org.apache.avro.io.JsonDecoder; +import org.apache.avro.io.BinaryDecoder; +import org.apache.pulsar.client.api.schema.Field; +import org.apache.pulsar.client.api.schema.GenericRecord; +import org.apache.pulsar.client.api.schema.GenericRecordBuilder; +import org.apache.pulsar.client.impl.schema.SchemaInfoImpl; +import org.apache.pulsar.client.impl.schema.generic.GenericAvroSchema; +import org.apache.pulsar.common.schema.SchemaInfo; +import org.apache.pulsar.common.schema.SchemaType; +import java.io.IOException; +import java.nio.charset.StandardCharsets; +import java.util.HashMap; +import java.util.List; + +public class PulsarAvroSchemaUtil { + //////////////////////// + // Get an OSS Apache Avro schema from a string definition + public static org.apache.avro.Schema GetSchema_ApacheAvro(String avroSchemDef) { + return new org.apache.avro.Schema.Parser().parse(avroSchemDef); + } + + // Get an OSS Apache Avro schema record from a JSON string that matches a specific OSS Apache Avro schema + public static org.apache.avro.generic.GenericRecord GetGenericRecord_ApacheAvro(org.apache.avro.Schema schema, String jsonData) { + org.apache.avro.generic.GenericRecord record = null; + + try { + org.apache.avro.generic.GenericDatumReader reader; + reader = new org.apache.avro.generic.GenericDatumReader<>(schema); + + JsonDecoder decoder = DecoderFactory.get().jsonDecoder(schema, jsonData); + + record = reader.read(null, decoder); + } catch (IOException ioe) { + ioe.printStackTrace(); + } + + return record; + } + + // Get an OSS Apache Avro schema record from a byte array that matches a specific OSS Apache Avro schema + public static org.apache.avro.generic.GenericRecord GetGenericRecord_ApacheAvro(org.apache.avro.Schema schema, byte[] bytesData) { + org.apache.avro.generic.GenericRecord record = null; + + try { + org.apache.avro.generic.GenericDatumReader reader; + reader = new org.apache.avro.generic.GenericDatumReader<>(schema); + + BinaryDecoder decoder = DecoderFactory.get().binaryDecoder(bytesData, null); + + record = reader.read(null, decoder); + } catch (IOException ioe) { + ioe.printStackTrace(); + } + + return record; + } + + + //////////////////////// + // Get a Pulsar Avro schema from a string definition + public static GenericAvroSchema GetSchema_PulsarAvro(String schemaName, String avroSchemDef) { + SchemaInfo schemaInfo = SchemaInfoImpl.builder() + .schema(avroSchemDef.getBytes(StandardCharsets.UTF_8)) + .type(SchemaType.AVRO) + .properties(new HashMap<>()) + .name(schemaName) + .build(); + return new GenericAvroSchema(schemaInfo); + } + + // Get a Pulsar Avro record from an OSS Avro schema record, matching a specific Pulsar Avro schema + public static GenericRecord GetGenericRecord_PulsarAvro( + GenericAvroSchema pulsarGenericAvroSchema, + org.apache.avro.generic.GenericRecord apacheAvroGenericRecord) + { + GenericRecordBuilder recordBuilder = pulsarGenericAvroSchema.newRecordBuilder(); + + List fieldList = pulsarGenericAvroSchema.getFields(); + for (Field field : fieldList) { + String fieldName = field.getName(); + recordBuilder.set(fieldName, apacheAvroGenericRecord.get(fieldName)); + } + + return recordBuilder.build(); + } + + // Get a Pulsar Avro record (GenericRecord) from a JSON string that matches a specific Pulsar Avro schema + public static GenericRecord GetGenericRecord_PulsarAvro(GenericAvroSchema genericAvroSchema, String avroSchemDefStr, String jsonData) { + org.apache.avro.Schema avroSchema = GetSchema_ApacheAvro(avroSchemDefStr); + return GetGenericRecord_PulsarAvro(genericAvroSchema, avroSchema, jsonData); + } + + public static GenericRecord GetGenericRecord_PulsarAvro(GenericAvroSchema genericAvroSchema, org.apache.avro.Schema avroSchema, String jsonData) { + org.apache.avro.generic.GenericRecord apacheAvroRecord = GetGenericRecord_ApacheAvro(avroSchema, jsonData); + return GetGenericRecord_PulsarAvro(genericAvroSchema, apacheAvroRecord); + } + public static GenericRecord GetGenericRecord_PulsarAvro(String schemaName, String avroSchemDefStr, String jsonData) { + GenericAvroSchema genericAvroSchema = GetSchema_PulsarAvro(schemaName, avroSchemDefStr); + org.apache.avro.Schema avroSchema = GetSchema_ApacheAvro(avroSchemDefStr); + org.apache.avro.generic.GenericRecord apacheAvroRecord = GetGenericRecord_ApacheAvro(avroSchema, jsonData); + + return GetGenericRecord_PulsarAvro(genericAvroSchema, apacheAvroRecord); + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/PulsarClientConf.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/PulsarClientConf.java new file mode 100644 index 000000000..9fd56be8f --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/PulsarClientConf.java @@ -0,0 +1,323 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.util; + +import org.apache.commons.configuration2.Configuration; +import org.apache.commons.configuration2.FileBasedConfiguration; +import org.apache.commons.configuration2.PropertiesConfiguration; +import org.apache.commons.configuration2.builder.FileBasedConfigurationBuilder; +import org.apache.commons.configuration2.builder.fluent.Parameters; +import org.apache.commons.configuration2.ex.ConfigurationException; +import org.apache.commons.lang3.StringUtils; +import org.apache.commons.lang3.builder.ToStringBuilder; +import org.apache.logging.log4j.LogManager; +import org.apache.logging.log4j.Logger; + +import java.io.File; +import java.io.IOException; +import java.util.HashMap; +import java.util.Iterator; +import java.util.Map; + +public class PulsarClientConf { + + private final static Logger logger = LogManager.getLogger(PulsarClientConf.class); + + private String canonicalFilePath = ""; + + public static final String SCHEMA_CONF_PREFIX = "schema"; + public static final String CLIENT_CONF_PREFIX = "client"; + public static final String PRODUCER_CONF_PREFIX = "producer"; + public static final String CONSUMER_CONF_PREFIX = "consumer"; + public static final String READER_CONF_PREFIX = "reader"; + private final Map schemaConfMapRaw = new HashMap<>(); + private final Map clientConfMapRaw = new HashMap<>(); + + // "Raw" map is what is read from the config properties file + // "Tgt" map is what is really needed in the Pulsar producer/consumer/reader API + private final Map producerConfMapRaw = new HashMap<>(); + private final Map producerConfMapTgt = new HashMap<>(); + + private final Map consumerConfMapRaw = new HashMap<>(); + private final Map consumerConfMapTgt = new HashMap<>(); + + private final Map readerConfMapRaw = new HashMap<>(); + private final Map readerConfMapTgt = new HashMap<>(); + + public PulsarClientConf(String fileName) { + + ////////////////// + // Read related Pulsar client configuration settings from a file + readRawConfFromFile(fileName); + + + ////////////////// + // Ignores the following Pulsar client/producer/consumer configurations since + // they need to be specified either as the NB CLI parameters or as the NB yaml + // OpTemplate parameters. + clientConfMapRaw.remove("brokerServiceUrl"); + clientConfMapRaw.remove("webServiceUrl"); + + + ////////////////// + // Convert the raw configuration map () to the required map () + producerConfMapTgt.putAll(PulsarConfConverter.convertRawProducerConf(producerConfMapRaw)); + consumerConfMapTgt.putAll(PulsarConfConverter.convertRawConsumerConf(consumerConfMapRaw)); + // TODO: Reader API is not disabled at the moment. Revisit when needed + } + + + public void readRawConfFromFile(String fileName) { + File file = new File(fileName); + + try { + canonicalFilePath = file.getCanonicalPath(); + + Parameters params = new Parameters(); + + FileBasedConfigurationBuilder builder = + new FileBasedConfigurationBuilder(PropertiesConfiguration.class) + .configure(params.properties() + .setFileName(fileName)); + + Configuration config = builder.getConfiguration(); + + for (Iterator it = config.getKeys(); it.hasNext(); ) { + String confKey = it.next(); + String confVal = config.getProperty(confKey).toString(); + + if (!StringUtils.isBlank(confVal)) { + + // Get schema specific configuration settings, removing "schema." prefix + if (StringUtils.startsWith(confKey, SCHEMA_CONF_PREFIX)) { + schemaConfMapRaw.put(confKey.substring(SCHEMA_CONF_PREFIX.length() + 1), confVal); + } + // Get client connection specific configuration settings, removing "client." prefix + // <<< https://pulsar.apache.org/docs/reference-configuration/#client >>> + else if (StringUtils.startsWith(confKey, CLIENT_CONF_PREFIX)) { + clientConfMapRaw.put(confKey.substring(CLIENT_CONF_PREFIX.length() + 1), confVal); + } + // Get producer specific configuration settings, removing "producer." prefix + // <<< https://pulsar.apache.org/docs/client-libraries-java/#configure-producer >>> + else if (StringUtils.startsWith(confKey, PRODUCER_CONF_PREFIX)) { + producerConfMapRaw.put(confKey.substring(PRODUCER_CONF_PREFIX.length() + 1), confVal); + } + // Get consumer specific configuration settings, removing "consumer." prefix + // <<< https://pulsar.apache.org/docs/client-libraries-java/#configure-consumer >>> + else if (StringUtils.startsWith(confKey, CONSUMER_CONF_PREFIX)) { + consumerConfMapRaw.put(confKey.substring(CONSUMER_CONF_PREFIX.length() + 1), confVal); + } + // Get reader specific configuration settings, removing "reader." prefix + // <<< https://pulsar.apache.org/docs/2.10.x/client-libraries-java/#configure-reader >>> + else if (StringUtils.startsWith(confKey, READER_CONF_PREFIX)) { + readerConfMapRaw.put(confKey.substring(READER_CONF_PREFIX.length() + 1), confVal); + } + } + } + } catch (IOException ioe) { + logger.error("Can't read the specified config properties file!"); + ioe.printStackTrace(); + } catch (ConfigurationException cex) { + logger.error("Error loading configuration items from the specified config properties file: " + canonicalFilePath); + cex.printStackTrace(); + } + } + + + public Map getSchemaConfMapRaw() { return this.schemaConfMapRaw; } + public Map getClientConfMapRaw() { return this.clientConfMapRaw; } + public Map getProducerConfMapRaw() { return this.producerConfMapRaw; } + public Map getProducerConfMapTgt() { return this.producerConfMapTgt; } + public Map getConsumerConfMapRaw() { return this.consumerConfMapRaw; } + public Map getConsumerConfMapTgt() { return this.consumerConfMapTgt; } + public Map getReaderConfMapRaw() { return this.readerConfMapRaw; } + public Map getReaderConfMapTgt() { return this.readerConfMapTgt; } + + + public String toString() { + return new ToStringBuilder(this). + append("schemaConfMapRaw", schemaConfMapRaw.toString()). + append("clientConfMapRaw", clientConfMapRaw.toString()). + append("producerConfMapRaw", producerConfMapRaw.toString()). + append("consumerConfMapRaw", consumerConfMapRaw.toString()). + append("readerConfMapRaw", readerConfMapRaw.toString()). + toString(); + } + + ////////////////// + // Get Schema related config + public boolean hasSchemaConfKey(String key) { + if (key.contains(SCHEMA_CONF_PREFIX)) + return schemaConfMapRaw.containsKey(key.substring(SCHEMA_CONF_PREFIX.length() + 1)); + else + return schemaConfMapRaw.containsKey(key); + } + public String getSchemaConfValue(String key) { + if (key.contains(SCHEMA_CONF_PREFIX)) + return schemaConfMapRaw.get(key.substring(SCHEMA_CONF_PREFIX.length()+1)); + else + return schemaConfMapRaw.get(key); + } + + + ////////////////// + // Get Pulsar client related config + public String getClientConfValue(String key) { + if (key.contains(CLIENT_CONF_PREFIX)) + return clientConfMapRaw.get(key.substring(CLIENT_CONF_PREFIX.length()+1)); + else + return clientConfMapRaw.get(key); + } + + + ////////////////// + // Get Pulsar producer related config + public Object getProducerConfValue(String key) { + if (key.contains(PRODUCER_CONF_PREFIX)) + return producerConfMapTgt.get(key.substring(PRODUCER_CONF_PREFIX.length()+1)); + else + return producerConfMapTgt.get(key); + } + // other producer helper functions ... + public String getProducerName() { + Object confValue = getProducerConfValue( + "producer." + PulsarAdapterUtil.PRODUCER_CONF_STD_KEY.producerName.label); + if (confValue == null) + return ""; + else + return confValue.toString(); + } + public String getProducerTopicName() { + Object confValue = getProducerConfValue( + "producer." + PulsarAdapterUtil.PRODUCER_CONF_STD_KEY.topicName); + if (confValue == null) + return ""; + else + return confValue.toString(); + } + + + ////////////////// + // Get Pulsar consumer related config + public String getConsumerConfValue(String key) { + if (key.contains(CONSUMER_CONF_PREFIX)) + return consumerConfMapRaw.get(key.substring(CONSUMER_CONF_PREFIX.length() + 1)); + else + return consumerConfMapRaw.get(key); + } + // Other consumer helper functions ... + public String getConsumerTopicNames() { + String confValue = getConsumerConfValue( + "consumer." + PulsarAdapterUtil.CONSUMER_CONF_STD_KEY.topicNames.label); + if (confValue == null) + return ""; + else + return confValue.toString(); + } + public String getConsumerTopicPattern() { + Object confValue = getConsumerConfValue( + "consumer." + PulsarAdapterUtil.CONSUMER_CONF_STD_KEY.topicsPattern.label); + if (confValue == null) + return ""; + else + return confValue.toString(); + } + public String getConsumerSubscriptionName() { + Object confValue = getConsumerConfValue( + "consumer." + PulsarAdapterUtil.CONSUMER_CONF_STD_KEY.subscriptionName.label); + if (confValue == null) + return ""; + else + return confValue.toString(); + } + public String getConsumerSubscriptionType() { + Object confValue = getConsumerConfValue( + "consumer." + PulsarAdapterUtil.CONSUMER_CONF_STD_KEY.subscriptionType.label); + if (confValue == null) + return ""; + else + return confValue.toString(); + } + public String getConsumerName() { + Object confValue = getConsumerConfValue( + "consumer." + PulsarAdapterUtil.CONSUMER_CONF_STD_KEY.consumerName.label); + if (confValue == null) + return ""; + else + return confValue.toString(); + } + // NOTE: Below are not a standard Pulsar consumer configuration parameter as + // listed in "https://pulsar.apache.org/docs/en/client-libraries-java/#configure-consumer" + // They're custom-made configuration properties for NB pulsar driver consumer. + public int getConsumerTimeoutSeconds() { + Object confValue = getConsumerConfValue( + "consumer." + PulsarAdapterUtil.CONSUMER_CONF_CUSTOM_KEY.timeout.label); + if (confValue == null) + return -1; // infinite + else + return Integer.parseInt(confValue.toString()); + } + + ////////////////// + // Get Pulsar reader related config + public boolean hasReaderConfKey(String key) { + if (key.contains(READER_CONF_PREFIX)) + return readerConfMapRaw.containsKey(key.substring(READER_CONF_PREFIX.length() + 1)); + else + return readerConfMapRaw.containsKey(key); + } + public Object getReaderConfValue(String key) { + if (key.contains(READER_CONF_PREFIX)) + return readerConfMapRaw.get(key.substring(READER_CONF_PREFIX.length() + 1)); + else + return readerConfMapRaw.get(key); + } + public void setReaderConfValue(String key, String value) { + if (key.contains(READER_CONF_PREFIX)) + readerConfMapRaw.put(key.substring(READER_CONF_PREFIX.length() + 1), value); + else + readerConfMapRaw.put(key, value); + } + // Other reader helper functions ... + public String getReaderTopicName() { + Object confValue = getReaderConfValue( + "reader." + PulsarAdapterUtil.READER_CONF_STD_KEY.topicName.label); + if (confValue == null) + return ""; + else + return confValue.toString(); + } + public String getReaderName() { + Object confValue = getReaderConfValue( + "reader." + PulsarAdapterUtil.READER_CONF_STD_KEY.readerName.label); + if (confValue == null) + return ""; + else + return confValue.toString(); + } + // NOTE: Below are not a standard Pulsar reader configuration parameter as + // listed in "https://pulsar.apache.org/docs/en/client-libraries-java/#reader" + // They're custom-made configuration properties for NB pulsar driver reader. + public String getStartMsgPosStr() { + Object confValue = getReaderConfValue( + "reader." + PulsarAdapterUtil.READER_CONF_CUSTOM_KEY.startMessagePos.label); + if (confValue == null) + return ""; + else + return confValue.toString(); + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/PulsarConfConverter.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/PulsarConfConverter.java new file mode 100644 index 000000000..8024cb39e --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/PulsarConfConverter.java @@ -0,0 +1,367 @@ +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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 io.nosqlbench.adapter.pulsar.util; + +import com.fasterxml.jackson.databind.ObjectMapper; +import io.nosqlbench.adapter.pulsar.exception.PulsarAdapterInvalidParamException; +import org.apache.commons.lang3.StringUtils; +import org.apache.commons.lang3.math.NumberUtils; +import org.apache.pulsar.client.api.CompressionType; +import org.apache.pulsar.client.api.DeadLetterPolicy; +import org.apache.pulsar.client.api.RedeliveryBackoff; +import org.apache.pulsar.client.impl.MultiplierRedeliveryBackoff; + +import java.util.ArrayList; +import java.util.HashMap; +import java.util.List; +import java.util.Map; + +public class PulsarConfConverter { + + // <<< https://pulsar.apache.org/docs/client-libraries-java/#configure-producer >>> + private final static Map validPulsarProducerConfKeyTypeMap = Map.ofEntries( + Map.entry("topicName", "String"), + Map.entry("producerName","String"), + Map.entry("sendTimeoutMs","long"), + Map.entry("blockIfQueueFull","boolean"), + Map.entry("maxPendingMessages","int"), + Map.entry("maxPendingMessagesAcrossPartitions","int"), + Map.entry("messageRoutingMode","MessageRoutingMode"), + Map.entry("hashingScheme","HashingScheme"), + Map.entry("cryptoFailureAction","ProducerCryptoFailureAction"), + Map.entry("batchingMaxPublishDelayMicros","long"), + Map.entry("batchingMaxMessages","int"), + Map.entry("batchingEnabled","boolean"), + Map.entry("chunkingEnabled","boolean"), + Map.entry("compressionType","CompressionType"), + Map.entry("initialSubscriptionName","string") + ); + public static Map convertRawProducerConf(Map pulsarProducerConfMapRaw) { + Map producerConfObjMap = new HashMap<>(); + setConfObjMapForPrimitives(producerConfObjMap, pulsarProducerConfMapRaw, validPulsarProducerConfKeyTypeMap); + + /** + * Non-primitive type processing for Pulsar producer configuration items + */ + // TODO: Skip the following Pulsar configuration items for now because they're not really + // needed in the NB S4J testing at the moment. Add support for them when needed. + // * messageRoutingMode + // * hashingScheme + // * cryptoFailureAction + + // "compressionType" has value type "CompressionType" + // - expecting the following values: 'LZ4', 'ZLIB', 'ZSTD', 'SNAPPY' + String confKeyName = "compressionType"; + String confVal = pulsarProducerConfMapRaw.get(confKeyName); + String expectedVal = "(LZ4|ZLIB|ZSTD|SNAPPY)"; + + if (StringUtils.isNotBlank(confVal)) { + if (StringUtils.equalsAnyIgnoreCase(confVal, "LZ4", "ZLIB", "ZSTD", "SNAPPY")) { + CompressionType compressionType = CompressionType.NONE; + + switch (StringUtils.upperCase(confVal)) { + case "LZ4": + compressionType = CompressionType.LZ4; + case "ZLIB": + compressionType = CompressionType.ZLIB; + case "ZSTD": + compressionType = CompressionType.ZSTD; + case "SNAPPY": + compressionType = CompressionType.SNAPPY; + } + + producerConfObjMap.put(confKeyName, compressionType); + } else { + throw new PulsarAdapterInvalidParamException( + getInvalidConfValStr(confKeyName, confVal, "producer", expectedVal)); + } + } + + return producerConfObjMap; + } + + + // https://pulsar.apache.org/docs/client-libraries-java/#configure-consumer + private final static Map validPulsarConsumerConfKeyTypeMap = Map.ofEntries( + Map.entry("topicNames", "Set"), + Map.entry("topicsPattern","Pattern"), + Map.entry("subscriptionName","String"), + Map.entry("subscriptionType","SubscriptionType"), + Map.entry("receiverQueueSize","int"), + Map.entry("acknowledgementsGroupTimeMicros","long"), + Map.entry("negativeAckRedeliveryDelayMicros","long"), + Map.entry("maxTotalReceiverQueueSizeAcrossPartitions","int"), + Map.entry("consumerName","String"), + Map.entry("ackTimeoutMillis","long"), + Map.entry("tickDurationMillis","long"), + Map.entry("priorityLevel","int"), + Map.entry("cryptoFailureAction","ConsumerCryptoFailureAction"), + Map.entry("properties","SortedMap"), + Map.entry("readCompacted","boolean"), + Map.entry("subscriptionInitialPosition", "SubscriptionInitialPosition"), + Map.entry("patternAutoDiscoveryPeriod", "int"), + Map.entry("regexSubscriptionMode", "RegexSubscriptionMode"), + Map.entry("deadLetterPolicy", "DeadLetterPolicy"), + Map.entry("autoUpdatePartitions", "boolean"), + Map.entry("replicateSubscriptionState", "boolean"), + Map.entry("negativeAckRedeliveryBackoff", "RedeliveryBackoff"), + Map.entry("ackTimeoutRedeliveryBackoff", "RedeliveryBackoff"), + Map.entry("autoAckOldestChunkedMessageOnQueueFull", "boolean"), + Map.entry("maxPendingChunkedMessage", "int"), + Map.entry("expireTimeOfIncompleteChunkedMessageMillis", "long") + ); + public static Map convertRawConsumerConf(Map pulsarConsumerConfMapRaw) { + Map consumerConfObjMap = new HashMap<>(); + setConfObjMapForPrimitives(consumerConfObjMap, pulsarConsumerConfMapRaw, validPulsarConsumerConfKeyTypeMap); + + /** + * Non-primitive type processing for Pulsar consumer configuration items + */ + // NOTE: The following non-primitive type configuration items are excluded since + // they'll be handled in PulsarBasedOpDispenser.getConsumer() method directly + // * topicNames + // * topicPattern + // * subscriptionType + + + // TODO: Skip the following Pulsar configuration items for now because they're not really + // needed in the NB S4J testing right now. Add the support for them when needed. + // * subscriptionInitialPosition + // * regexSubscriptionMode + // * cryptoFailureAction + + + // "properties" has value type "SortedMap" + // - expecting the value string has the format: a JSON string that includes a set of key/value pairs + String confKeyName = "properties"; + String confVal = pulsarConsumerConfMapRaw.get(confKeyName); + String expectedVal = "{\"property1\":\"value1\", \"property2\":\"value2\"}, ..."; + + ObjectMapper mapper = new ObjectMapper(); + + if (StringUtils.isNotBlank(confVal)) { + try { + Map consumerProperties = mapper.readValue(confVal, Map.class); + + // Empty map value is considered as no value + if (!consumerProperties.isEmpty()) { + consumerConfObjMap.put(confKeyName, consumerProperties); + } + + } catch (Exception e) { + throw new PulsarAdapterInvalidParamException( + getInvalidConfValStr(confKeyName, confVal, "consumer", expectedVal)); + } + } + + // "deadLetterPolicy" + // - expecting the value is a JSON string has the format: + // {"maxRedeliverCount":"","deadLetterTopic":"","initialSubscriptionName":""} + confKeyName = "deadLetterPolicy"; + confVal = pulsarConsumerConfMapRaw.get(confKeyName); + expectedVal = "{" + + "\"maxRedeliverCount\":\"\"," + + "\"deadLetterTopic\":\"\"," + + "\"initialSubscriptionName\":\"\"}"; + + if (StringUtils.isNotBlank(confVal)) { + try { + Map dlqPolicyMap = mapper.readValue(confVal, Map.class); + + // Empty map value is considered as no value + if (!dlqPolicyMap.isEmpty()) { + boolean valid = true; + + // The JSON key must be one of "maxRedeliverCount", "deadLetterTopic", "initialSubscriptionName" + for (String key : dlqPolicyMap.keySet()) { + if (!StringUtils.equalsAnyIgnoreCase(key, + "maxRedeliverCount", "deadLetterTopic", "initialSubscriptionName")) { + valid = false; + break; + } + } + + // DLQ.maxRedeliverCount is mandatory + if (valid && !dlqPolicyMap.containsKey("maxRedeliverCount")) { + valid = false; + } + + String maxRedeliverCountStr = dlqPolicyMap.get("maxRedeliverCount"); + if (!NumberUtils.isCreatable(maxRedeliverCountStr)) { + valid = false; + } + + if (valid) { + DeadLetterPolicy deadLetterPolicy = DeadLetterPolicy.builder() + .maxRedeliverCount(NumberUtils.toInt(maxRedeliverCountStr)) + .deadLetterTopic(dlqPolicyMap.get("deadLetterTopic")) + .initialSubscriptionName(dlqPolicyMap.get("initialSubscriptionName")) + .build(); + + consumerConfObjMap.put(confKeyName, deadLetterPolicy); + } else { + throw new PulsarAdapterInvalidParamException( + getInvalidConfValStr(confKeyName, confVal, "consumer", expectedVal)); + } + } + } catch (Exception e) { + throw new PulsarAdapterInvalidParamException( + getInvalidConfValStr(confKeyName, confVal, "consumer", expectedVal)); + } + } + + // "negativeAckRedeliveryBackoff" or "ackTimeoutRedeliveryBackoff" + // - expecting the value is a JSON string has the format: + // {"minDelayMs":"", "maxDelayMs":"", "multiplier":""} + String[] redeliveryBackoffConfigSet = {"negativeAckRedeliveryBackoff", "ackTimeoutRedeliveryBackoff"}; + expectedVal = "{" + + "\"minDelayMs\":\"\"," + + "\"maxDelayMs\":\"\"," + + "\"multiplier\":\"\"}"; + + for (String confKey : redeliveryBackoffConfigSet) { + confVal = pulsarConsumerConfMapRaw.get(confKey); + + if (StringUtils.isNotBlank(confVal)) { + try { + Map redliveryBackoffMap = mapper.readValue(confVal, Map.class); + + // Empty map value is considered as no value + if (! redliveryBackoffMap.isEmpty()) { + boolean valid = true; + + // The JSON key must be one of "maxRedeliverCount", "deadLetterTopic", "initialSubscriptionName" + for (String key : redliveryBackoffMap.keySet()) { + if (!StringUtils.equalsAnyIgnoreCase(key, + "minDelayMs", "maxDelayMs", "multiplier")) { + valid = false; + break; + } + } + + String minDelayMsStr = redliveryBackoffMap.get("minDelayMs"); + String maxDelayMsStr = redliveryBackoffMap.get("maxDelayMs"); + String multiplierStr = redliveryBackoffMap.get("multiplier"); + + if ((StringUtils.isNotBlank(minDelayMsStr) && !NumberUtils.isCreatable(minDelayMsStr)) || + (StringUtils.isNotBlank(maxDelayMsStr) && !NumberUtils.isCreatable(maxDelayMsStr)) || + (StringUtils.isNotBlank(multiplierStr) && !NumberUtils.isCreatable(multiplierStr))) { + valid = false; + } + + if (valid) { + RedeliveryBackoff redeliveryBackoff = MultiplierRedeliveryBackoff.builder() + .minDelayMs(NumberUtils.toLong(minDelayMsStr)) + .maxDelayMs(NumberUtils.toLong(maxDelayMsStr)) + .multiplier(NumberUtils.toDouble(multiplierStr)) + .build(); + + consumerConfObjMap.put(confKey, redeliveryBackoff); + + } else { + throw new PulsarAdapterInvalidParamException( + getInvalidConfValStr(confKey, confVal, "consumer", expectedVal)); + } + } + + } catch (Exception e) { + throw new PulsarAdapterInvalidParamException( + getInvalidConfValStr(confKey, confVal, "consumer", expectedVal)); + } + } + } + + return consumerConfObjMap; + } + + + // Utility function + // - get configuration key names by the value type + private static List getConfKeyNameByValueType(Map confKeyTypeMap, String tgtValType) { + ArrayList confKeyNames = new ArrayList<>(); + + for (Map.Entry entry: confKeyTypeMap.entrySet()) { + if (StringUtils.equalsIgnoreCase(entry.getValue().toString(), tgtValType)) { + confKeyNames.add(entry.getKey().toString()); + } + } + + return confKeyNames; + } + + // Conversion from Map to Map for configuration items with primitive + // value types + private static void setConfObjMapForPrimitives( + Map tgtConfObjMap, + Map srcConfMapRaw, + Map validConfKeyTypeMap) + { + List confKeyList = new ArrayList<>(); + + // All configuration items with "String" as the value type + confKeyList = getConfKeyNameByValueType(validConfKeyTypeMap, "String"); + for (String confKey : confKeyList) { + if (srcConfMapRaw.containsKey(confKey)) { + String confVal = srcConfMapRaw.get(confKey); + if (StringUtils.isNotBlank(confVal)) { + tgtConfObjMap.put(confKey, confVal); + } + } + } + + // All configuration items with "long" as the value type + confKeyList = getConfKeyNameByValueType(validConfKeyTypeMap, "long"); + for (String confKey : confKeyList) { + if (srcConfMapRaw.containsKey(confKey)) { + String confVal = srcConfMapRaw.get(confKey); + if (StringUtils.isNotBlank(confVal)) { + tgtConfObjMap.put(confKey, Long.valueOf(confVal)); + } + } + } + + // All configuration items with "int" as the value type + confKeyList = getConfKeyNameByValueType(validConfKeyTypeMap, "int"); + for (String confKey : confKeyList) { + if (srcConfMapRaw.containsKey(confKey)) { + String confVal = srcConfMapRaw.get(confKey); + if (StringUtils.isNotBlank(confVal)) { + tgtConfObjMap.put(confKey, Integer.valueOf(confVal)); + } + } + } + + // All configuration items with "boolean" as the value type + confKeyList = getConfKeyNameByValueType(validConfKeyTypeMap, "boolean"); + for (String confKey : confKeyList) { + if (srcConfMapRaw.containsKey(confKey)) { + String confVal = srcConfMapRaw.get(confKey); + if (StringUtils.isNotBlank(confVal)) { + tgtConfObjMap.put(confKey, Boolean.valueOf(confVal)); + } + } + } + + // TODO: So far the above primitive types should be good enough. + // Add support for other types when needed + } + + private static String getInvalidConfValStr(String confKey, String confVal, String configCategory, String expectedVal) { + return "Incorrect value \"" + confVal + "\" for Pulsar " + configCategory + + " configuration item of \"" + confKey + "\". Expecting the following value (format): " + expectedVal; + } +} diff --git a/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/ReceivedMessageSequenceTracker.java b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/ReceivedMessageSequenceTracker.java new file mode 100644 index 000000000..f929ab25a --- /dev/null +++ b/adapter-pulsar/src/main/java/io/nosqlbench/adapter/pulsar/util/ReceivedMessageSequenceTracker.java @@ -0,0 +1,169 @@ +package io.nosqlbench.adapter.pulsar.util; + +/* + * Copyright (c) 2022 nosqlbench + * + * Licensed 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. + */ + + +import com.codahale.metrics.Counter; + +import java.util.Iterator; +import java.util.SortedSet; +import java.util.TreeSet; + +/** + * Detects message loss, message duplication and out-of-order message delivery + * based on a monotonic sequence number that each received message contains. + *

+ * Out-of-order messages are detected with a maximum look behind of 1000 sequence number entries. + * This is currently defined as a constant, {@link ReceivedMessageSequenceTracker#DEFAULT_MAX_TRACK_OUT_OF_ORDER_SEQUENCE_NUMBERS}. + */ +public class ReceivedMessageSequenceTracker implements AutoCloseable { + private static final int DEFAULT_MAX_TRACK_OUT_OF_ORDER_SEQUENCE_NUMBERS = 1000; + private static final int DEFAULT_MAX_TRACK_SKIPPED_SEQUENCE_NUMBERS = 1000; + // message out-of-sequence error counter + private final Counter msgErrOutOfSeqCounter; + // duplicate message error counter + private final Counter msgErrDuplicateCounter; + // message loss error counter + private final Counter msgErrLossCounter; + private final SortedSet pendingOutOfSeqNumbers; + private final int maxTrackOutOfOrderSequenceNumbers; + private final SortedSet skippedSeqNumbers; + private final int maxTrackSkippedSequenceNumbers; + private long expectedNumber = -1; + + public ReceivedMessageSequenceTracker(Counter msgErrOutOfSeqCounter, Counter msgErrDuplicateCounter, Counter msgErrLossCounter) { + this(msgErrOutOfSeqCounter, msgErrDuplicateCounter, msgErrLossCounter, + DEFAULT_MAX_TRACK_OUT_OF_ORDER_SEQUENCE_NUMBERS, DEFAULT_MAX_TRACK_SKIPPED_SEQUENCE_NUMBERS); + } + + public ReceivedMessageSequenceTracker(Counter msgErrOutOfSeqCounter, Counter msgErrDuplicateCounter, Counter msgErrLossCounter, + int maxTrackOutOfOrderSequenceNumbers, int maxTrackSkippedSequenceNumbers) { + this.msgErrOutOfSeqCounter = msgErrOutOfSeqCounter; + this.msgErrDuplicateCounter = msgErrDuplicateCounter; + this.msgErrLossCounter = msgErrLossCounter; + this.maxTrackOutOfOrderSequenceNumbers = maxTrackOutOfOrderSequenceNumbers; + this.maxTrackSkippedSequenceNumbers = maxTrackSkippedSequenceNumbers; + this.pendingOutOfSeqNumbers = new TreeSet<>(); + this.skippedSeqNumbers = new TreeSet<>(); + } + + /** + * Notifies the tracker about a received sequence number + * + * @param sequenceNumber the sequence number of the received message + */ + public void sequenceNumberReceived(long sequenceNumber) { + if (expectedNumber == -1) { + expectedNumber = sequenceNumber + 1; + return; + } + + if (sequenceNumber < expectedNumber) { + if (skippedSeqNumbers.remove(sequenceNumber)) { + // late out-of-order delivery was detected + // decrease the loss counter + msgErrLossCounter.dec(); + // increment the out-of-order counter + msgErrOutOfSeqCounter.inc(); + } else { + msgErrDuplicateCounter.inc(); + } + return; + } + + boolean messagesSkipped = false; + if (sequenceNumber > expectedNumber) { + if (pendingOutOfSeqNumbers.size() == maxTrackOutOfOrderSequenceNumbers) { + messagesSkipped = processLowestPendingOutOfSequenceNumber(); + } + if (!pendingOutOfSeqNumbers.add(sequenceNumber)) { + msgErrDuplicateCounter.inc(); + } + } else { + // sequenceNumber == expectedNumber + expectedNumber++; + } + processPendingOutOfSequenceNumbers(messagesSkipped); + cleanUpTooFarBehindOutOfSequenceNumbers(); + } + + private boolean processLowestPendingOutOfSequenceNumber() { + // remove the lowest pending out of sequence number + Long lowestOutOfSeqNumber = pendingOutOfSeqNumbers.first(); + pendingOutOfSeqNumbers.remove(lowestOutOfSeqNumber); + if (lowestOutOfSeqNumber > expectedNumber) { + // skip the expected number ahead to the number after the lowest sequence number + // increment the counter with the amount of sequence numbers that got skipped + // keep track of the skipped sequence numbers to detect late out-of-order message delivery + for (long l = expectedNumber; l < lowestOutOfSeqNumber; l++) { + msgErrLossCounter.inc(); + skippedSeqNumbers.add(l); + if (skippedSeqNumbers.size() > maxTrackSkippedSequenceNumbers) { + skippedSeqNumbers.remove(skippedSeqNumbers.first()); + } + } + expectedNumber = lowestOutOfSeqNumber + 1; + return true; + } else { + msgErrLossCounter.inc(); + } + return false; + } + + private void processPendingOutOfSequenceNumbers(boolean messagesSkipped) { + // check if there are previously received out-of-order sequence number that have been received + while (pendingOutOfSeqNumbers.remove(expectedNumber)) { + expectedNumber++; + if (!messagesSkipped) { + msgErrOutOfSeqCounter.inc(); + } + } + } + + private void cleanUpTooFarBehindOutOfSequenceNumbers() { + // remove sequence numbers that are too far behind + for (Iterator iterator = pendingOutOfSeqNumbers.iterator(); iterator.hasNext(); ) { + Long number = iterator.next(); + if (number < expectedNumber - maxTrackOutOfOrderSequenceNumbers) { + msgErrLossCounter.inc(); + iterator.remove(); + } else { + break; + } + } + } + + /** + * Handles the possible pending out of sequence numbers. Mainly needed in unit tests to assert the + * counter values. + */ + @Override + public void close() { + while (!pendingOutOfSeqNumbers.isEmpty()) { + processPendingOutOfSequenceNumbers(processLowestPendingOutOfSequenceNumber()); + } + } + + public int getMaxTrackOutOfOrderSequenceNumbers() { + return maxTrackOutOfOrderSequenceNumbers; + } + + public int getMaxTrackSkippedSequenceNumbers() { + return maxTrackSkippedSequenceNumbers; + } +} diff --git a/adapter-pulsar/src/main/resources/admin_namespace.yaml b/adapter-pulsar/src/main/resources/admin_namespace.yaml new file mode 100644 index 000000000..4fd9b18ac --- /dev/null +++ b/adapter-pulsar/src/main/resources/admin_namespace.yaml @@ -0,0 +1,16 @@ +bindings: + # 20 topics: 10 tenants, 2 namespaces/tenant + tenant: Mod(20); Div(2); ToString(); Prefix("tnt") + namespace: Mod(2); ToString(); Prefix("ns") + +params: + async_api: "false" + admin_delop: "false" + +blocks: + admin-namespace-block: + tags: + phase: admin-namespace + ops: + op1: + AdminNamespace: "{tenant}/{namespace}" diff --git a/adapter-pulsar/src/main/resources/admin_tenant.yaml b/adapter-pulsar/src/main/resources/admin_tenant.yaml new file mode 100644 index 000000000..865643fde --- /dev/null +++ b/adapter-pulsar/src/main/resources/admin_tenant.yaml @@ -0,0 +1,17 @@ +bindings: + # 10 tenants + tenant: Mod(10); ToString(); Prefix("tnt") + +params: + async_api: "false" + admin_delop: "false" + +blocks: + admin-tenant-block: + tags: + phase: admin-tenant + ops: + op1: + AdminTopic: "{tenant}" + admin_roles: "" + allowed_clusters: "" diff --git a/adapter-pulsar/src/main/resources/admin_topic.yaml b/adapter-pulsar/src/main/resources/admin_topic.yaml new file mode 100644 index 000000000..6e30ad2bf --- /dev/null +++ b/adapter-pulsar/src/main/resources/admin_topic.yaml @@ -0,0 +1,19 @@ +bindings: + # 100 topics: 10 tenants, 2 namespaces/tenant, 5 topics/namespace + tenant: Mod(100); Div(10); ToString(); Prefix("tnt") + namespace: Mod(10); Div(5); ToString(); Prefix("ns") + topic: Mod(5); ToString(); Prefix("tp") + +params: + async_api: "false" + admin_delop: "false" + +blocks: + admin-topic-block: + tags: + phase: admin-topic + ops: + op1: + AdminTopic: "{tenant}/{namespace}/{topic}" + enable_partition: "false" + partition_num: "5" diff --git a/adapter-pulsar/src/main/resources/bindingtest.yaml b/adapter-pulsar/src/main/resources/bindingtest.yaml new file mode 100644 index 000000000..e687f5fa4 --- /dev/null +++ b/adapter-pulsar/src/main/resources/bindingtest.yaml @@ -0,0 +1,4 @@ +bindings: + tenant: Mod(100); Div(10); ToString(); Prefix("tnt") + namespace: Mod(10); Div(5); ToString(); Prefix("ns") + topic: Mod(5); ToString(); Prefix("tp") diff --git a/adapter-pulsar/src/main/resources/config.properties b/adapter-pulsar/src/main/resources/config.properties new file mode 100644 index 000000000..9cc804c1f --- /dev/null +++ b/adapter-pulsar/src/main/resources/config.properties @@ -0,0 +1,52 @@ +9### Schema related configurations - schema.xxx +# valid types: +# - primitive type (https://pulsar.apache.org/docs/en/schema-understand/#primitive-type) +# - keyvalue (https://pulsar.apache.org/docs/en/schema-understand/#keyvalue) +# - strut (complex type) (https://pulsar.apache.org/docs/en/schema-understand/#struct) +# avro, json, protobuf +# +# TODO: as a starting point, only supports the following types +# 1) primitive types, including bytearray (byte[]) which is default, for messages without schema +# 2) Avro for messages with schema +#schema.key.type=avro +#schema.key.definition=file:///Users/yabinmeng/DataStax/MyNBMain/nosqlbench/adapter-pulsar/src/main/resources/iot-key-example.avsc +#schema.type=avro +#schema.definition=file:///Users/yabinmeng/DataStax/MyNBMain/nosqlbench/adapter-pulsar/src/main/resources/iot-value-example.avsc +schema.type= +schema.definition= + + +### Pulsar client related configurations - client.xxx +# http://pulsar.apache.org/docs/en/client-libraries-java/#client +client.connectionTimeoutMs=5000 +client.authPluginClassName=org.apache.pulsar.client.impl.auth.AuthenticationToken +# Cluster admin +client.authParams= +client.tlsAllowInsecureConnection=true + + +### Producer related configurations (global) - producer.xxx +# http://pulsar.apache.org/docs/en/client-libraries-java/#configure-producer +producer.producerName= +producer.topicName= +producer.sendTimeoutMs= +producer.blockIfQueueFull=true + + +### Consumer related configurations (global) - consumer.xxx +# http://pulsar.apache.org/docs/en/client-libraries-java/#configure-consumer +consumer.topicNames= +consumer.topicsPattern= +consumer.subscriptionName= +consumer.subscriptionType= +consumer.consumerName= +consumer.receiverQueueSize= + + +### Reader related configurations (global) - reader.xxx +# https://pulsar.apache.org/docs/en/client-libraries-java/#reader +# - valid Pos: earliest, latest, custom::file://// +reader.topicName= +reader.receiverQueueSize= +reader.readerName= +reader.startMessagePos=earliest diff --git a/adapter-pulsar/src/main/resources/iot-key-example.avsc b/adapter-pulsar/src/main/resources/iot-key-example.avsc new file mode 100644 index 000000000..f36b52bc3 --- /dev/null +++ b/adapter-pulsar/src/main/resources/iot-key-example.avsc @@ -0,0 +1,9 @@ +{ + "type": "record", + "name": "IotSensorKey", + "namespace": "TestNS", + "fields" : [ + {"name": "Location", "type": "string"}, + {"name": "WellID", "type": "string"} + ] +} diff --git a/adapter-pulsar/src/main/resources/iot-value-example.avsc b/adapter-pulsar/src/main/resources/iot-value-example.avsc new file mode 100644 index 000000000..20bb894fd --- /dev/null +++ b/adapter-pulsar/src/main/resources/iot-value-example.avsc @@ -0,0 +1,11 @@ +{ + "type": "record", + "name": "IotSensor", + "namespace": "TestNS", + "fields" : [ + {"name": "SensorID", "type": "string"}, + {"name": "SensorType", "type": "string"}, + {"name": "ReadingTime", "type": "string"}, + {"name": "ReadingValue", "type": "float"} + ] +} diff --git a/adapter-pulsar/src/main/resources/msg_proc_avro.yaml b/adapter-pulsar/src/main/resources/msg_proc_avro.yaml new file mode 100644 index 000000000..bfc07a176 --- /dev/null +++ b/adapter-pulsar/src/main/resources/msg_proc_avro.yaml @@ -0,0 +1,41 @@ +bindings: + # message key and value + mykey: NumberNameToString() + location: Cities(); + well_id: ToUUID();ToString(); + sensor_id: ToUUID();ToString(); + reading_time: ToDateTime(); + reading_value: ToFloat(100); + +# document level parameters that apply to all Pulsar client types: +params: + async_api: "true" + +blocks: + msg-produce-block: + tags: + phase: msg-send + ops: + op1: + MessageProduce: "tnt0/ns0/tp1" + msg_key: | + { + "Location": "{location}", + "WellID": "{well_id}" + } + msg_value: | + { + "SensorID": "{sensor_id}", + "SensorType": "Temperature", + "ReadingTime": "{reading_time}", + "ReadingValue": {reading_value} + } + + msg-consume-block: + tags: + phase: msg-recv + ops: + op1: + MessageConsume: "tnt0/ns0/tp0" + subscription_name: "mynbsub" +# subscription_type: "shared" diff --git a/adapter-pulsar/src/main/resources/msg_proc_kvraw.yaml b/adapter-pulsar/src/main/resources/msg_proc_kvraw.yaml new file mode 100644 index 000000000..4aea8f44a --- /dev/null +++ b/adapter-pulsar/src/main/resources/msg_proc_kvraw.yaml @@ -0,0 +1,34 @@ +bindings: + # message key, property and value + mykey: NumberNameToString() + int_prop_val: ToString(); Prefix("IntProp_") + text_prop_val: AlphaNumericString(5); Prefix("TextProp_") + myvalue: AlphaNumericString(20) + +# document level parameters that apply to all Pulsar client types: +params: + async_api: "true" + +blocks: + msg-produce-block: + tags: + phase: msg-send + ops: + op1: + MessageProduce: "tnt0/ns0/tp0" + msg_key: "{mykey}" + msg_prop: | + { + "prop1": "{int_prop_val}", + "prop2": "{text_prop_val}" + } + msg_value: "{myvalue}" + + msg-consume-block: + tags: + phase: msg-recv + ops: + op1: + MessageConsume: "tnt0/ns0/tp0" + subscription_name: "mynbsub" +# subscription_type: "shared" diff --git a/adapter-pulsar/src/main/resources/pulsar.md b/adapter-pulsar/src/main/resources/pulsar.md new file mode 100644 index 000000000..586fecbb2 --- /dev/null +++ b/adapter-pulsar/src/main/resources/pulsar.md @@ -0,0 +1 @@ +<< to be added ... >> diff --git a/adapters-api/src/main/java/io/nosqlbench/engine/api/activityimpl/BaseOpDispenser.java b/adapters-api/src/main/java/io/nosqlbench/engine/api/activityimpl/BaseOpDispenser.java index 740ef3348..7bfc2d60a 100644 --- a/adapters-api/src/main/java/io/nosqlbench/engine/api/activityimpl/BaseOpDispenser.java +++ b/adapters-api/src/main/java/io/nosqlbench/engine/api/activityimpl/BaseOpDispenser.java @@ -39,7 +39,7 @@ public abstract class BaseOpDispenser implements OpDispenser private final String name; protected final DriverAdapter adapter; - private boolean instrument; + protected boolean instrument; private Histogram resultSizeHistogram; private Timer successTimer; private Timer errorTimer; @@ -83,12 +83,16 @@ public abstract class BaseOpDispenser implements OpDispenser @Override public abstract T apply(long cycle); + protected String getDefaultMetricsPrefix(ParsedOp pop) { + return pop.getStaticConfigOr("alias", "UNKNOWN") + "-" + pop.getName() + "--"; + } + private void configureInstrumentation(ParsedOp pop) { this.instrument = pop.takeStaticConfigOr("instrument", false); if (instrument) { - this.successTimer = ActivityMetrics.timer(pop.getStaticConfigOr("alias", "UNKNOWN") + "-" + pop.getName() + "--success"); - this.errorTimer = ActivityMetrics.timer(pop.getStaticConfigOr("alias", "UNKNOWN") + "-" + pop.getName() + "--error"); - this.resultSizeHistogram = ActivityMetrics.histogram(pop.getStaticConfigOr("alias", "UNKNOWN") + "-" + pop.getName() + "--resultset-size"); + this.successTimer = ActivityMetrics.timer(getDefaultMetricsPrefix(pop) + "success"); + this.errorTimer = ActivityMetrics.timer(getDefaultMetricsPrefix(pop) + "error"); + this.resultSizeHistogram = ActivityMetrics.histogram(getDefaultMetricsPrefix(pop) + "resultset-size"); } } diff --git a/driver-jms/src/main/java/io/nosqlbench/driver/jms/JmsAction.java b/driver-jms/src/main/java/io/nosqlbench/driver/jms/JmsAction.java deleted file mode 100644 index 3a3ef9b2a..000000000 --- a/driver-jms/src/main/java/io/nosqlbench/driver/jms/JmsAction.java +++ /dev/null @@ -1,84 +0,0 @@ -/* - * Copyright (c) 2022 nosqlbench - * - * Licensed 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 io.nosqlbench.driver.jms; - -import com.codahale.metrics.Timer; -import io.nosqlbench.driver.jms.ops.JmsOp; -import io.nosqlbench.engine.api.activityapi.core.SyncAction; -import io.nosqlbench.engine.api.activityapi.errorhandling.modular.ErrorDetail; -import org.apache.logging.log4j.LogManager; -import org.apache.logging.log4j.Logger; - -import java.util.function.LongFunction; - -public class JmsAction implements SyncAction { - - private final static Logger logger = LogManager.getLogger(JmsAction.class); - - private final JmsActivity activity; - private final int slot; - - int maxTries; - - public JmsAction(JmsActivity activity, int slot) { - this.activity = activity; - this.slot = slot; - this.maxTries = activity.getActivityDef().getParams().getOptionalInteger("maxtries").orElse(10); - } - - @Override - public void init() { } - - @Override - public int runCycle(long cycle) { - // let's fail the action if some async operation failed - activity.failOnAsyncOperationFailure(); - - long start = System.nanoTime(); - - JmsOp jmsOp; - try (Timer.Context ctx = activity.getBindTimer().time()) { - LongFunction readyJmsOp = activity.getSequencer().apply(cycle); - jmsOp = readyJmsOp.apply(cycle); - } catch (Exception bindException) { - // if diagnostic mode ... - activity.getErrorhandler().handleError(bindException, cycle, 0); - throw new RuntimeException( - "while binding request in cycle " + cycle + ": " + bindException.getMessage(), bindException - ); - } - - for (int i = 0; i < maxTries; i++) { - Timer.Context ctx = activity.getExecuteTimer().time(); - try { - // it is up to the jmsOp to call Context#close when the activity is executed - // this allows us to track time for async operations - jmsOp.run(ctx::close); - break; - } catch (RuntimeException err) { - ErrorDetail errorDetail = activity - .getErrorhandler() - .handleError(err, cycle, System.nanoTime() - start); - if (!errorDetail.isRetryable()) { - break; - } - } - } - - return 0; - } -} diff --git a/driver-jms/src/main/java/io/nosqlbench/driver/jms/JmsActivity.java b/driver-jms/src/main/java/io/nosqlbench/driver/jms/JmsActivity.java deleted file mode 100644 index 2da3a440c..000000000 --- a/driver-jms/src/main/java/io/nosqlbench/driver/jms/JmsActivity.java +++ /dev/null @@ -1,179 +0,0 @@ -/* - * Copyright (c) 2022 nosqlbench - * - * Licensed 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 io.nosqlbench.driver.jms; - -import com.codahale.metrics.Counter; -import com.codahale.metrics.Histogram; -import com.codahale.metrics.Timer; -import com.datastax.oss.pulsar.jms.PulsarConnectionFactory; -import io.nosqlbench.driver.jms.conn.JmsConnInfo; -import io.nosqlbench.driver.jms.conn.JmsPulsarConnInfo; -import io.nosqlbench.driver.jms.ops.JmsOp; -import io.nosqlbench.driver.jms.util.JmsUtil; -import io.nosqlbench.driver.jms.util.PulsarConfig; -import io.nosqlbench.engine.api.activityapi.errorhandling.modular.NBErrorHandler; -import io.nosqlbench.engine.api.activityapi.planning.OpSequence; -import io.nosqlbench.api.engine.activityimpl.ActivityDef; -import io.nosqlbench.engine.api.activityimpl.OpDispenser; -import io.nosqlbench.engine.api.activityimpl.SimpleActivity; -import io.nosqlbench.api.engine.metrics.ActivityMetrics; -import org.apache.commons.lang3.StringUtils; - -import javax.jms.Destination; -import javax.jms.JMSContext; -import java.util.Optional; -import java.util.concurrent.ConcurrentHashMap; - - -public class JmsActivity extends SimpleActivity { - - private final ConcurrentHashMap jmsDestinations = new ConcurrentHashMap<>(); - - private String jmsProviderType; - private JmsConnInfo jmsConnInfo; - - private JMSContext jmsContext; - - private OpSequence> sequence; - private volatile Throwable asyncOperationFailure; - private NBErrorHandler errorhandler; - - private Timer bindTimer; - private Timer executeTimer; - private Counter bytesCounter; - private Histogram messagesizeHistogram; - - public JmsActivity(ActivityDef activityDef) { - super(activityDef); - } - - @Override - public void initActivity() { - super.initActivity(); - - // default JMS type: Pulsar - // - currently this is the only supported JMS provider - jmsProviderType = - activityDef.getParams() - .getOptionalString(JmsUtil.JMS_PROVIDER_TYPE_KEY_STR) - .orElse(JmsUtil.JMS_PROVIDER_TYPES.PULSAR.label); - - // "Pulsar" as the JMS provider - if (StringUtils.equalsIgnoreCase(jmsProviderType, JmsUtil.JMS_PROVIDER_TYPES.PULSAR.label )) { - - String webSvcUrl = - activityDef.getParams() - .getOptionalString(JmsUtil.JMS_PULSAR_PROVIDER_WEB_URL_KEY_STR) - .orElse("http://localhost:8080"); - String pulsarSvcUrl = - activityDef.getParams() - .getOptionalString(JmsUtil.JMS_PULSAR_PROVIDER_SVC_URL_KEY_STR) - .orElse("pulsar://localhost:6650"); - - if (StringUtils.isAnyBlank(webSvcUrl, pulsarSvcUrl)) { - throw new RuntimeException("For \"" + JmsUtil.JMS_PROVIDER_TYPES.PULSAR.label + "\" type, " + - "\"" + JmsUtil.JMS_PULSAR_PROVIDER_WEB_URL_KEY_STR + "\" and " + - "\"" + JmsUtil.JMS_PULSAR_PROVIDER_SVC_URL_KEY_STR + "\" parameters are manadatory!"); - } - - // Check if extra Pulsar config. file is in place - // - default file: "pulsar_config.properties" under the current directory - String pulsarCfgFile = - activityDef.getParams() - .getOptionalString(JmsUtil.JMS_PULSAR_PROVIDER_CFG_FILE_KEY_STR) - .orElse(JmsUtil.JMS_PULSAR_PROVIDER_DFT_CFG_FILE_NAME); - - PulsarConfig pulsarConfig = new PulsarConfig(pulsarCfgFile); - - jmsConnInfo = new JmsPulsarConnInfo(jmsProviderType, webSvcUrl, pulsarSvcUrl, pulsarConfig); - } - else { - throw new RuntimeException("Unsupported JMS driver type : " + jmsProviderType); - } - - PulsarConnectionFactory factory; - factory = new PulsarConnectionFactory(jmsConnInfo.getJmsConnConfig()); - this.jmsContext = factory.createContext(); - - bindTimer = ActivityMetrics.timer(activityDef, "bind", this.getHdrDigits()); - executeTimer = ActivityMetrics.timer(activityDef, "execute", this.getHdrDigits()); - bytesCounter = ActivityMetrics.counter(activityDef, "bytes"); - messagesizeHistogram = ActivityMetrics.histogram(activityDef, "messagesize", this.getHdrDigits()); - - if (StringUtils.equalsIgnoreCase(jmsProviderType, JmsUtil.JMS_PROVIDER_TYPES.PULSAR.label )) { - this.sequence = createOpSequence((ot) -> new ReadyPulsarJmsOp(ot, this), false, Optional.empty()); - } - - setDefaultsFromOpSequence(sequence); - onActivityDefUpdate(activityDef); - - this.errorhandler = new NBErrorHandler( - () -> activityDef.getParams().getOptionalString("errors").orElse("stop"), - this::getExceptionMetrics - ); - } - - private static String buildCacheKey(String... keyParts) { - return String.join("::", keyParts); - } - - /** - * If the JMS destination that corresponds to a topic exists, reuse it; Otherwise, create it - */ - public Destination getOrCreateJmsDestination(String jmsDestinationType, String destName) { - String destinationCacheKey = buildCacheKey(jmsDestinationType, destName); - Destination destination = jmsDestinations.get(destinationCacheKey); - - if ( destination == null ) { - // TODO: should we match Persistent/Non-peristent JMS Delivery mode with - // Pulsar Persistent/Non-prsistent topic? - if (StringUtils.equalsIgnoreCase(jmsDestinationType, JmsUtil.JMS_DESTINATION_TYPES.QUEUE.label)) { - destination = jmsContext.createQueue(destName); - } else if (StringUtils.equalsIgnoreCase(jmsDestinationType, JmsUtil.JMS_DESTINATION_TYPES.TOPIC.label)) { - destination = jmsContext.createTopic(destName); - } - - jmsDestinations.put(destinationCacheKey, destination); - } - - return destination; - } - - @Override - public synchronized void onActivityDefUpdate(ActivityDef activityDef) { super.onActivityDefUpdate(activityDef); } - public OpSequence> getSequencer() { return sequence; } - - public String getJmsProviderType() { return jmsProviderType; } - public JmsConnInfo getJmsConnInfo() { return jmsConnInfo; } - public JMSContext getJmsContext() { return jmsContext; } - - public Timer getBindTimer() { return bindTimer; } - public Timer getExecuteTimer() { return this.executeTimer; } - public Counter getBytesCounter() { return bytesCounter; } - public Histogram getMessagesizeHistogram() { return messagesizeHistogram; } - - public NBErrorHandler getErrorhandler() { return errorhandler; } - - public void failOnAsyncOperationFailure() { - if (asyncOperationFailure != null) { - throw new RuntimeException(asyncOperationFailure); - } - } - public void asyncOperationFailed(Throwable ex) { - this.asyncOperationFailure = ex; - } -} diff --git a/driver-jms/src/main/java/io/nosqlbench/driver/jms/JmsActivityType.java b/driver-jms/src/main/java/io/nosqlbench/driver/jms/JmsActivityType.java deleted file mode 100644 index b964516ea..000000000 --- a/driver-jms/src/main/java/io/nosqlbench/driver/jms/JmsActivityType.java +++ /dev/null @@ -1,48 +0,0 @@ -/* - * Copyright (c) 2022 nosqlbench - * - * Licensed 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 io.nosqlbench.driver.jms; - -import io.nosqlbench.engine.api.activityapi.core.Action; -import io.nosqlbench.engine.api.activityapi.core.ActionDispenser; -import io.nosqlbench.engine.api.activityapi.core.ActivityType; -import io.nosqlbench.api.engine.activityimpl.ActivityDef; -import io.nosqlbench.nb.annotations.Service; - -@Service(value = ActivityType.class, selector = "jms") -public class JmsActivityType implements ActivityType { - @Override - public ActionDispenser getActionDispenser(JmsActivity activity) { - return new PulsarJmsActionDispenser(activity); - } - - @Override - public JmsActivity getActivity(ActivityDef activityDef) { - return new JmsActivity(activityDef); - } - - private static class PulsarJmsActionDispenser implements ActionDispenser { - private final JmsActivity activity; - public PulsarJmsActionDispenser(JmsActivity activity) { - this.activity = activity; - } - - @Override - public Action getAction(int slot) { - return new JmsAction(activity, slot); - } - } -} diff --git a/driver-jms/src/main/java/io/nosqlbench/driver/jms/ReadyJmsOp.java b/driver-jms/src/main/java/io/nosqlbench/driver/jms/ReadyJmsOp.java deleted file mode 100644 index bd532033f..000000000 --- a/driver-jms/src/main/java/io/nosqlbench/driver/jms/ReadyJmsOp.java +++ /dev/null @@ -1,78 +0,0 @@ -/* - * Copyright (c) 2022 nosqlbench - * - * Licensed 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 io.nosqlbench.driver.jms; - -import io.nosqlbench.driver.jms.ops.JmsOp; -import io.nosqlbench.driver.jms.util.JmsUtil; -import io.nosqlbench.engine.api.activityconfig.yaml.OpTemplate; -import io.nosqlbench.engine.api.activityimpl.BaseOpDispenser; -import io.nosqlbench.engine.api.templating.CommandTemplate; -import org.apache.commons.lang3.BooleanUtils; - -import java.util.function.LongFunction; - -abstract public class ReadyJmsOp extends BaseOpDispenser { - - protected final OpTemplate optpl; - protected final CommandTemplate cmdTpl; - protected final JmsActivity jmsActivity; - - protected final String stmtOpType; - protected LongFunction asyncApiFunc; - protected LongFunction jmsDestinationTypeFunc; - - protected final LongFunction opFunc; - - public ReadyJmsOp(OpTemplate opTemplate, JmsActivity jmsActivity) { - super(opTemplate); - this.optpl = opTemplate; - this.cmdTpl = new CommandTemplate(optpl); - this.jmsActivity = jmsActivity; - - if (!cmdTpl.containsKey("optype") || !cmdTpl.isStatic("optype")) { - throw new RuntimeException("Statement parameter \"optype\" must be static and have a valid value!"); - } - this.stmtOpType = cmdTpl.getStatic("optype"); - - // Global/Doc-level parameter: async_api - if (cmdTpl.containsKey(JmsUtil.ASYNC_API_KEY_STR)) { - if (cmdTpl.isStatic(JmsUtil.ASYNC_API_KEY_STR)) { - boolean value = BooleanUtils.toBoolean(cmdTpl.getStatic(JmsUtil.ASYNC_API_KEY_STR)); - this.asyncApiFunc = (l) -> value; - } else { - throw new RuntimeException("\"" + JmsUtil.ASYNC_API_KEY_STR + "\" parameter cannot be dynamic!"); - } - } - - // Global/Doc-level parameter: jms_desitation_type - // - queue: point-to-point - // - topic: pub/sub - if (cmdTpl.containsKey(JmsUtil.JMS_DESTINATION_TYPE_KEY_STR)) { - if (cmdTpl.isStatic(JmsUtil.JMS_DESTINATION_TYPE_KEY_STR)) { - jmsDestinationTypeFunc = (l) -> cmdTpl.getStatic(JmsUtil.JMS_DESTINATION_TYPE_KEY_STR); - } else { - throw new RuntimeException("\"" + JmsUtil.JMS_DESTINATION_TYPE_KEY_STR + "\" parameter cannot be dynamic!"); - } - } - - this.opFunc = resolveJms(); - } - - public JmsOp apply(long value) { return opFunc.apply(value); } - - abstract LongFunction resolveJms(); -} diff --git a/driver-jms/src/main/java/io/nosqlbench/driver/jms/ReadyPulsarJmsOp.java b/driver-jms/src/main/java/io/nosqlbench/driver/jms/ReadyPulsarJmsOp.java deleted file mode 100644 index 72c397a97..000000000 --- a/driver-jms/src/main/java/io/nosqlbench/driver/jms/ReadyPulsarJmsOp.java +++ /dev/null @@ -1,264 +0,0 @@ -/* - * Copyright (c) 2022 nosqlbench - * - * Licensed 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 io.nosqlbench.driver.jms; - -import io.nosqlbench.driver.jms.ops.JmsMsgReadMapper; -import io.nosqlbench.driver.jms.ops.JmsMsgSendMapper; -import io.nosqlbench.driver.jms.ops.JmsOp; -import io.nosqlbench.driver.jms.util.JmsHeaderLongFunc; -import io.nosqlbench.driver.jms.util.JmsUtil; -import io.nosqlbench.engine.api.activityconfig.yaml.OpTemplate; -import org.apache.commons.lang3.BooleanUtils; -import org.apache.commons.lang3.StringUtils; -import org.apache.commons.lang3.math.NumberUtils; - -import javax.jms.DeliveryMode; -import javax.jms.Destination; -import javax.jms.JMSRuntimeException; -import javax.jms.Message; -import java.util.Arrays; -import java.util.HashMap; -import java.util.Map; -import java.util.function.LongFunction; -import java.util.stream.Collectors; - -public class ReadyPulsarJmsOp extends ReadyJmsOp { - - public ReadyPulsarJmsOp(OpTemplate opTemplate, JmsActivity jmsActivity) { - super(opTemplate, jmsActivity); - } - - public LongFunction resolveJms() { - // Global/Doc-level parameter: topic_uri - LongFunction topicUriFunc = (l) -> null; - if (cmdTpl.containsKey(JmsUtil.PULSAR_JMS_TOPIC_URI_KEY_STR)) { - if (cmdTpl.isStatic(JmsUtil.PULSAR_JMS_TOPIC_URI_KEY_STR)) { - topicUriFunc = (l) -> cmdTpl.getStatic(JmsUtil.PULSAR_JMS_TOPIC_URI_KEY_STR); - } else { - topicUriFunc = (l) -> cmdTpl.getDynamic(JmsUtil.PULSAR_JMS_TOPIC_URI_KEY_STR, l); - } - } - - // Global: JMS destination - LongFunction jmsDestinationFunc; - try { - LongFunction finalTopicUriFunc = topicUriFunc; - jmsDestinationFunc = (l) -> jmsActivity.getOrCreateJmsDestination( - jmsDestinationTypeFunc.apply(l), - finalTopicUriFunc.apply(l)); - } - catch (JMSRuntimeException ex) { - throw new RuntimeException("Unable to create JMS destination!"); - } - - if (StringUtils.equalsIgnoreCase(stmtOpType, JmsUtil.OP_TYPES.MSG_SEND.label)) { - return resolveMsgSend(asyncApiFunc, jmsDestinationFunc); - } else if (StringUtils.equalsIgnoreCase(stmtOpType, JmsUtil.OP_TYPES.MSG_READ.label)) { - return resolveMsgRead(asyncApiFunc, jmsDestinationFunc); - } else { - throw new RuntimeException("Unsupported JMS operation type"); - } - } - - private LongFunction resolveMsgSend( - LongFunction async_api_func, - LongFunction jmsDestinationFunc - ) { - JmsHeaderLongFunc jmsHeaderLongFunc = new JmsHeaderLongFunc(); - - // JMS header: delivery mode - LongFunction msgDeliveryModeFunc = (l) -> DeliveryMode.PERSISTENT; - if (cmdTpl.containsKey(JmsUtil.JMS_MSG_HEADER_KEYS.DELIVERY_MODE.label)) { - if (cmdTpl.isStatic(JmsUtil.JMS_MSG_HEADER_KEYS.DELIVERY_MODE.label)) { - msgDeliveryModeFunc = (l) -> NumberUtils.toInt(cmdTpl.getStatic(JmsUtil.JMS_MSG_HEADER_KEYS.DELIVERY_MODE.label)); - } - else { - msgDeliveryModeFunc = (l) -> NumberUtils.toInt(cmdTpl.getDynamic(JmsUtil.JMS_MSG_HEADER_KEYS.DELIVERY_MODE.label, l)); - } - } - jmsHeaderLongFunc.setDeliveryModeFunc(msgDeliveryModeFunc); - - // JMS header: message priority - LongFunction msgPriorityFunc = (l) -> Message.DEFAULT_PRIORITY; - if (cmdTpl.containsKey(JmsUtil.JMS_MSG_HEADER_KEYS.PRIORITY.label)) { - if (cmdTpl.isStatic(JmsUtil.JMS_MSG_HEADER_KEYS.PRIORITY.label)) { - msgPriorityFunc = (l) -> NumberUtils.toInt(cmdTpl.getStatic(JmsUtil.JMS_MSG_HEADER_KEYS.PRIORITY.label)); - } - else { - msgPriorityFunc = (l) -> NumberUtils.toInt(cmdTpl.getDynamic(JmsUtil.JMS_MSG_HEADER_KEYS.PRIORITY.label, l)); - } - } - jmsHeaderLongFunc.setMsgPriorityFunc(msgPriorityFunc); - - // JMS header: message TTL - LongFunction msgTtlFunc = (l) -> Message.DEFAULT_TIME_TO_LIVE; - if (cmdTpl.containsKey(JmsUtil.JMS_MSG_HEADER_KEYS.TTL.label)) { - if (cmdTpl.isStatic(JmsUtil.JMS_MSG_HEADER_KEYS.TTL.label)) { - msgTtlFunc = (l) -> NumberUtils.toLong(cmdTpl.getStatic(JmsUtil.JMS_MSG_HEADER_KEYS.TTL.label)); - } - else { - msgTtlFunc = (l) -> NumberUtils.toLong(cmdTpl.getDynamic(JmsUtil.JMS_MSG_HEADER_KEYS.TTL.label, l)); - } - } - jmsHeaderLongFunc.setMsgTtlFunc(msgTtlFunc); - - // JMS header: message delivery delay - LongFunction msgDeliveryDelayFunc = (l) -> Message.DEFAULT_DELIVERY_DELAY; - if (cmdTpl.containsKey(JmsUtil.JMS_MSG_HEADER_KEYS.DELIVERY_DELAY.label)) { - if (cmdTpl.isStatic(JmsUtil.JMS_MSG_HEADER_KEYS.DELIVERY_DELAY.label)) { - msgDeliveryDelayFunc = (l) -> NumberUtils.toLong(cmdTpl.getStatic(JmsUtil.JMS_MSG_HEADER_KEYS.DELIVERY_DELAY.label)); - } - else { - msgDeliveryDelayFunc = (l) -> NumberUtils.toLong(cmdTpl.getDynamic(JmsUtil.JMS_MSG_HEADER_KEYS.DELIVERY_DELAY.label, l)); - } - } - jmsHeaderLongFunc.setMsgDeliveryDelayFunc(msgDeliveryDelayFunc); - - // JMS header: disable message timestamp - LongFunction disableMsgTimestampFunc = (l) -> false; - if (cmdTpl.containsKey(JmsUtil.JMS_MSG_HEADER_KEYS.DISABLE_TIMESTAMP.label)) { - if (cmdTpl.isStatic(JmsUtil.JMS_MSG_HEADER_KEYS.DISABLE_TIMESTAMP.label)) { - disableMsgTimestampFunc = (l) -> BooleanUtils.toBoolean(cmdTpl.getStatic(JmsUtil.JMS_MSG_HEADER_KEYS.DISABLE_TIMESTAMP.label)); - } - else { - disableMsgTimestampFunc = (l) -> BooleanUtils.toBoolean(cmdTpl.getDynamic(JmsUtil.JMS_MSG_HEADER_KEYS.DISABLE_TIMESTAMP.label, l)); - } - } - jmsHeaderLongFunc.setDisableMsgTimestampFunc(disableMsgTimestampFunc); - - // JMS header: disable message ID - LongFunction disableMsgIdFunc = (l) -> false; - if (cmdTpl.containsKey(JmsUtil.JMS_MSG_HEADER_KEYS.DISABLE_ID.label)) { - if (cmdTpl.isStatic(JmsUtil.JMS_MSG_HEADER_KEYS.DISABLE_ID.label)) { - disableMsgIdFunc = (l) -> BooleanUtils.toBoolean(cmdTpl.getStatic(JmsUtil.JMS_MSG_HEADER_KEYS.DISABLE_ID.label)); - } - else { - disableMsgIdFunc = (l) -> BooleanUtils.toBoolean(cmdTpl.getDynamic(JmsUtil.JMS_MSG_HEADER_KEYS.DISABLE_ID.label, l)); - } - } - jmsHeaderLongFunc.setDisableMsgIdFunc(disableMsgIdFunc); - - // JMS message properties - String jmsMsgPropertyListStr = ""; - if (cmdTpl.containsKey(JmsUtil.JMS_PRODUCER_MSG_PROPERTY_KEY_STR)) { - if (cmdTpl.isStatic(JmsUtil.JMS_PRODUCER_MSG_PROPERTY_KEY_STR)) { - jmsMsgPropertyListStr = cmdTpl.getStatic(JmsUtil.JMS_PRODUCER_MSG_PROPERTY_KEY_STR); - } else { - throw new RuntimeException("\"" + JmsUtil.JMS_PRODUCER_MSG_PROPERTY_KEY_STR + "\" parameter cannot be dynamic!"); - } - } - - Map jmsMsgProperties = new HashMap<>(); - if ( !StringUtils.isEmpty(jmsMsgPropertyListStr) ) { - jmsMsgProperties = Arrays.stream(jmsMsgPropertyListStr.split(";")) - .map(s -> s.split("=", 2)) - .collect(Collectors.toMap(a -> a[0], a -> a.length > 1 ? a[1] : "")); - } - - LongFunction msgBodyFunc; - if (cmdTpl.containsKey(JmsUtil.JMS_PRODUCER_MSG_BODY_KEY_STR)) { - if (cmdTpl.isStatic(JmsUtil.JMS_PRODUCER_MSG_BODY_KEY_STR)) { - msgBodyFunc = (l) -> cmdTpl.getStatic(JmsUtil.JMS_PRODUCER_MSG_BODY_KEY_STR); - } else if (cmdTpl.isDynamic(JmsUtil.JMS_PRODUCER_MSG_BODY_KEY_STR)) { - msgBodyFunc = (l) -> cmdTpl.getDynamic(JmsUtil.JMS_PRODUCER_MSG_BODY_KEY_STR, l); - } else { - msgBodyFunc = (l) -> null; - } - } else { - throw new RuntimeException("JMS message send:: \"msg_body\" field must be specified!"); - } - - return new JmsMsgSendMapper( - jmsActivity, - async_api_func, - jmsDestinationFunc, - jmsHeaderLongFunc, - jmsMsgProperties, - msgBodyFunc); - } - - private LongFunction resolveMsgRead( - LongFunction async_api_func, - LongFunction jmsDestinationFunc - ) { - // For Pulsar JMS, make "durable" as the default - LongFunction jmsConsumerDurableFunc = (l) -> true; - if (cmdTpl.containsKey(JmsUtil.JMS_CONSUMER_DURABLE_KEY_STR)) { - if (cmdTpl.isStatic(JmsUtil.JMS_CONSUMER_DURABLE_KEY_STR)) { - jmsConsumerDurableFunc = (l) -> BooleanUtils.toBoolean(cmdTpl.getStatic(JmsUtil.JMS_CONSUMER_DURABLE_KEY_STR)); - } else if (cmdTpl.isDynamic(JmsUtil.JMS_CONSUMER_DURABLE_KEY_STR)) { - jmsConsumerDurableFunc = (l) -> BooleanUtils.toBoolean(cmdTpl.getDynamic(JmsUtil.JMS_CONSUMER_DURABLE_KEY_STR, l)); - } - } - - LongFunction jmsConsumerSharedFunc = (l) -> true; - if (cmdTpl.containsKey(JmsUtil.JMS_CONSUMER_SHARED_KEY_STR)) { - if (cmdTpl.isStatic(JmsUtil.JMS_CONSUMER_SHARED_KEY_STR)) { - jmsConsumerSharedFunc = (l) -> BooleanUtils.toBoolean(cmdTpl.getStatic(JmsUtil.JMS_CONSUMER_SHARED_KEY_STR)); - } else if (cmdTpl.isDynamic(JmsUtil.JMS_CONSUMER_SHARED_KEY_STR)) { - jmsConsumerSharedFunc = (l) -> BooleanUtils.toBoolean(cmdTpl.getDynamic(JmsUtil.JMS_CONSUMER_SHARED_KEY_STR, l)); - } - } - - LongFunction jmsMsgSubscriptionFunc = (l) -> ""; - if (cmdTpl.containsKey(JmsUtil.JMS_CONSUMER_MSG_SUBSCRIPTIOn_KEY_STR)) { - if (cmdTpl.isStatic(JmsUtil.JMS_CONSUMER_MSG_SUBSCRIPTIOn_KEY_STR)) { - jmsMsgSubscriptionFunc = (l) -> cmdTpl.getStatic(JmsUtil.JMS_CONSUMER_MSG_SUBSCRIPTIOn_KEY_STR); - } else if (cmdTpl.isDynamic(JmsUtil.JMS_CONSUMER_MSG_SUBSCRIPTIOn_KEY_STR)) { - jmsMsgSubscriptionFunc = (l) -> cmdTpl.getDynamic(JmsUtil.JMS_CONSUMER_MSG_SUBSCRIPTIOn_KEY_STR, l); - } - } - - LongFunction jmsMsgReadSelectorFunc = (l) -> ""; - if (cmdTpl.containsKey(JmsUtil.JMS_CONSUMER_MSG_READ_SELECTOR_KEY_STR)) { - if (cmdTpl.isStatic(JmsUtil.JMS_CONSUMER_MSG_READ_SELECTOR_KEY_STR)) { - jmsMsgReadSelectorFunc = (l) -> cmdTpl.getStatic(JmsUtil.JMS_CONSUMER_MSG_READ_SELECTOR_KEY_STR); - } else if (cmdTpl.isDynamic(JmsUtil.JMS_CONSUMER_MSG_READ_SELECTOR_KEY_STR)) { - jmsMsgReadSelectorFunc = (l) -> cmdTpl.getDynamic(JmsUtil.JMS_CONSUMER_MSG_READ_SELECTOR_KEY_STR, l); - } - } - - LongFunction jmsMsgNoLocalFunc = (l) -> true; - if (cmdTpl.containsKey(JmsUtil.JMS_CONSUMER_MSG_NOLOCAL_KEY_STR)) { - if (cmdTpl.isStatic(JmsUtil.JMS_CONSUMER_MSG_NOLOCAL_KEY_STR)) { - jmsMsgNoLocalFunc = (l) -> BooleanUtils.toBoolean(cmdTpl.getStatic(JmsUtil.JMS_CONSUMER_MSG_NOLOCAL_KEY_STR)); - } else if (cmdTpl.isDynamic(JmsUtil.JMS_CONSUMER_MSG_NOLOCAL_KEY_STR)) { - jmsMsgNoLocalFunc = (l) -> BooleanUtils.toBoolean(cmdTpl.getDynamic(JmsUtil.JMS_CONSUMER_MSG_NOLOCAL_KEY_STR, l)); - } - } - - LongFunction jmsReadTimeoutFunc = (l) -> 0L; - if (cmdTpl.containsKey(JmsUtil.JMS_CONSUMER_READ_TIMEOUT_KEY_STR)) { - if (cmdTpl.isStatic(JmsUtil.JMS_CONSUMER_READ_TIMEOUT_KEY_STR)) { - jmsReadTimeoutFunc = (l) -> NumberUtils.toLong(cmdTpl.getStatic(JmsUtil.JMS_CONSUMER_READ_TIMEOUT_KEY_STR)); - } else if (cmdTpl.isDynamic(JmsUtil.JMS_CONSUMER_READ_TIMEOUT_KEY_STR)) { - jmsReadTimeoutFunc = (l) -> NumberUtils.toLong(cmdTpl.getDynamic(JmsUtil.JMS_CONSUMER_READ_TIMEOUT_KEY_STR, l)); - } - } - - return new JmsMsgReadMapper( - jmsActivity, - async_api_func, - jmsDestinationFunc, - jmsConsumerDurableFunc, - jmsConsumerSharedFunc, - jmsMsgSubscriptionFunc, - jmsMsgReadSelectorFunc, - jmsMsgNoLocalFunc, - jmsReadTimeoutFunc); - } -} diff --git a/driver-jms/src/main/java/io/nosqlbench/driver/jms/conn/JmsConnInfo.java b/driver-jms/src/main/java/io/nosqlbench/driver/jms/conn/JmsConnInfo.java deleted file mode 100644 index 738f83678..000000000 --- a/driver-jms/src/main/java/io/nosqlbench/driver/jms/conn/JmsConnInfo.java +++ /dev/null @@ -1,37 +0,0 @@ -/* - * Copyright (c) 2022 nosqlbench - * - * Licensed 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 io.nosqlbench.driver.jms.conn; - -import java.util.HashMap; -import java.util.Map; - -public class JmsConnInfo { - - protected final String jmsProviderType; - protected final Map jmsConnConfig; - - protected JmsConnInfo(String jmsProviderType) { - this.jmsProviderType = jmsProviderType; - this.jmsConnConfig = new HashMap<>(); - } - - public Map getJmsConnConfig() { return this.jmsConnConfig; } - public void resetJmsConnConfig() { this.jmsConnConfig.clear(); } - public void addJmsConnConfigItems(Map cfgItems) { this.jmsConnConfig.putAll(cfgItems); } - public void addJmsConnConfigItem(String key, Object value) { this.jmsConnConfig.put(key, value); } - public void removeJmsConnConfigItem(String key) { this.jmsConnConfig.remove(key); } -} diff --git a/driver-jms/src/main/java/io/nosqlbench/driver/jms/conn/JmsPulsarConnInfo.java b/driver-jms/src/main/java/io/nosqlbench/driver/jms/conn/JmsPulsarConnInfo.java deleted file mode 100644 index f1e09fe78..000000000 --- a/driver-jms/src/main/java/io/nosqlbench/driver/jms/conn/JmsPulsarConnInfo.java +++ /dev/null @@ -1,58 +0,0 @@ -/* - * Copyright (c) 2022 nosqlbench - * - * Licensed 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 io.nosqlbench.driver.jms.conn; - -import io.nosqlbench.driver.jms.util.PulsarConfig; - -import java.util.Map; - -public class JmsPulsarConnInfo extends JmsConnInfo { - - private final String webSvcUrl; - private final String pulsarSvcUrl; - private final PulsarConfig extraPulsarConfig; - - public JmsPulsarConnInfo(String jmsProviderType, String webSvcUrl, String pulsarSvcUrl, PulsarConfig pulsarConfig) { - super(jmsProviderType); - - this.webSvcUrl = webSvcUrl; - this.pulsarSvcUrl = pulsarSvcUrl; - this.extraPulsarConfig = pulsarConfig; - - this.addJmsConnConfigItem("webServiceUrl", this.webSvcUrl); - this.addJmsConnConfigItem("brokerServiceUrl", this.pulsarSvcUrl); - - Map clientCfgMap = this.extraPulsarConfig.getClientConfMap(); - if (!clientCfgMap.isEmpty()) { - this.addJmsConnConfigItems(clientCfgMap); - } - - Map producerCfgMap = this.extraPulsarConfig.getProducerConfMap(); - if (!producerCfgMap.isEmpty()) { - this.addJmsConnConfigItem("producerConfig", producerCfgMap); - } - - Map consumerCfgMap = this.extraPulsarConfig.getConsumerConfMap(); - if (!consumerCfgMap.isEmpty()) { - this.addJmsConnConfigItem("consumerConfig", consumerCfgMap); - } - } - - public String getWebSvcUrl() { return this.webSvcUrl; } - public String getPulsarSvcUrl() { return this.pulsarSvcUrl; } - public PulsarConfig getExtraPulsarConfig() { return this.extraPulsarConfig; } -} diff --git a/driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsMsgReadMapper.java b/driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsMsgReadMapper.java deleted file mode 100644 index 0108c0608..000000000 --- a/driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsMsgReadMapper.java +++ /dev/null @@ -1,88 +0,0 @@ -/* - * Copyright (c) 2022 nosqlbench - * - * Licensed 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 io.nosqlbench.driver.jms.ops; - -import io.nosqlbench.driver.jms.JmsActivity; - -import javax.jms.Destination; -import java.util.function.LongFunction; - -/** - * This maps a set of specifier functions to a pulsar operation. The pulsar operation contains - * enough state to define a pulsar operation such that it can be executed, measured, and possibly - * retried if needed. - * - * This function doesn't act *as* the operation. It merely maps the construction logic into - * a simple functional type, given the component functions. - * - * For additional parameterization, the command template is also provided. - */ -public class JmsMsgReadMapper extends JmsOpMapper { - - private final LongFunction jmsConsumerDurableFunc; - private final LongFunction jmsConsumerSharedFunc; - private final LongFunction jmsMsgSubscriptionFunc; - private final LongFunction jmsMsgReadSelectorFunc; - private final LongFunction jmsMsgNoLocalFunc; - private final LongFunction jmsReadTimeoutFunc; - - public JmsMsgReadMapper(JmsActivity jmsActivity, - LongFunction asyncApiFunc, - LongFunction jmsDestinationFunc, - LongFunction jmsConsumerDurableFunc, - LongFunction jmsConsumerSharedFunc, - LongFunction jmsMsgSubscriptionFunc, - LongFunction jmsMsgReadSelectorFunc, - LongFunction jmsMsgNoLocalFunc, - LongFunction jmsReadTimeoutFunc) { - super(jmsActivity, asyncApiFunc, jmsDestinationFunc); - - this.jmsConsumerDurableFunc = jmsConsumerDurableFunc; - this.jmsConsumerSharedFunc = jmsConsumerSharedFunc; - this.jmsMsgSubscriptionFunc = jmsMsgSubscriptionFunc; - this.jmsMsgReadSelectorFunc = jmsMsgReadSelectorFunc; - this.jmsMsgNoLocalFunc = jmsMsgNoLocalFunc; - this.jmsReadTimeoutFunc = jmsReadTimeoutFunc; - } - - @Override - public JmsOp apply(long value) { - boolean asyncApi = asyncApiFunc.apply(value); - Destination jmsDestination = jmsDestinationFunc.apply(value); - boolean jmsConsumerDurable = jmsConsumerDurableFunc.apply(value); - boolean jmsConsumerShared = jmsConsumerSharedFunc.apply(value); - String jmsMsgSubscription = jmsMsgSubscriptionFunc.apply(value); - String jmsMsgReadSelector = jmsMsgReadSelectorFunc.apply(value); - boolean jmsMsgNoLocal = jmsMsgNoLocalFunc.apply(value); - long jmsReadTimeout = jmsReadTimeoutFunc.apply(value); - - // Default to NO read timeout - if (jmsReadTimeout < 0) jmsReadTimeout = 0; - - return new JmsMsgReadOp( - jmsActivity, - asyncApi, - jmsDestination, - jmsConsumerDurable, - jmsConsumerShared, - jmsMsgSubscription, - jmsMsgReadSelector, - jmsMsgNoLocal, - jmsReadTimeout - ); - } -} diff --git a/driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsMsgReadOp.java b/driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsMsgReadOp.java deleted file mode 100644 index de92f73c5..000000000 --- a/driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsMsgReadOp.java +++ /dev/null @@ -1,130 +0,0 @@ -/* - * Copyright (c) 2022 nosqlbench - * - * Licensed 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 io.nosqlbench.driver.jms.ops; - -import com.codahale.metrics.Counter; -import com.codahale.metrics.Histogram; -import io.nosqlbench.driver.jms.JmsActivity; -import org.apache.logging.log4j.LogManager; -import org.apache.logging.log4j.Logger; - -import javax.jms.*; - -public class JmsMsgReadOp extends JmsTimeTrackOp { - - private final static Logger logger = LogManager.getLogger(JmsMsgReadOp.class); - - private final JmsActivity jmsActivity; - private final boolean asyncJmsOp; - private final Destination jmsDestination; - - private final JMSContext jmsContext; - private final JMSConsumer jmsConsumer; - private final boolean jmsConsumerDurable; - private final boolean jmsConsumerShared; - private final String jmsMsgSubscrption; - private final String jmsMsgReadSelector; - private final boolean jmsMsgNoLocal; - private final long jmsReadTimeout; - - private final Counter bytesCounter; - private final Histogram messagesizeHistogram; - - public JmsMsgReadOp(JmsActivity jmsActivity, - boolean asyncJmsOp, - Destination jmsDestination, - boolean jmsConsumerDurable, - boolean jmsConsumerShared, - String jmsMsgSubscrption, - String jmsMsgReadSelector, - boolean jmsMsgNoLocal, - long jmsReadTimeout) { - this.jmsActivity = jmsActivity; - this.asyncJmsOp = asyncJmsOp; - this.jmsDestination = jmsDestination; - this.jmsConsumerDurable = jmsConsumerDurable; - this.jmsConsumerShared = jmsConsumerShared; - this.jmsMsgReadSelector = jmsMsgReadSelector; - this.jmsMsgSubscrption = jmsMsgSubscrption; - this.jmsMsgNoLocal = jmsMsgNoLocal; - this.jmsReadTimeout = jmsReadTimeout; - - this.jmsContext = jmsActivity.getJmsContext(); - this.jmsConsumer = createJmsConsumer(); - - this.bytesCounter = jmsActivity.getBytesCounter(); - this.messagesizeHistogram = jmsActivity.getMessagesizeHistogram(); - } - - private JMSConsumer createJmsConsumer() { - JMSConsumer jmsConsumer; - - try { - if (jmsConsumerDurable) { - if (jmsConsumerShared) - jmsConsumer = jmsContext.createSharedDurableConsumer((Topic) jmsDestination, jmsMsgSubscrption, jmsMsgReadSelector); - else - jmsConsumer = jmsContext.createDurableConsumer((Topic) jmsDestination, jmsMsgSubscrption, jmsMsgReadSelector, jmsMsgNoLocal); - } else { - if (jmsConsumerShared) - jmsConsumer = jmsContext.createSharedConsumer((Topic) jmsDestination, jmsMsgSubscrption, jmsMsgReadSelector); - else - jmsConsumer = jmsContext.createConsumer(jmsDestination, jmsMsgReadSelector, jmsMsgNoLocal); - } - } - catch (InvalidDestinationRuntimeException invalidDestinationRuntimeException) { - throw new RuntimeException("Failed to create JMS consumer: invalid destination!"); - } - catch (InvalidSelectorRuntimeException invalidSelectorRuntimeException) { - throw new RuntimeException("Failed to create JMS consumer: invalid message selector!"); - } - catch (JMSRuntimeException jmsRuntimeException) { - jmsRuntimeException.printStackTrace(); - throw new RuntimeException("Failed to create JMS consumer: runtime internal error!"); - } - - // TODO: async consumer -// if (this.asyncJmsOp) { -// jmsConsumer.setMessageListener(); -// } - - return jmsConsumer; - } - - @Override - public void run() { - // FIXME: jmsReadTimeout being 0 behaves like receiveNoWait() instead of waiting indefinitley - Message receivedMsg = jmsConsumer.receive(jmsReadTimeout); - try { - if (receivedMsg != null) { - receivedMsg.acknowledge(); - byte[] receivedMsgBody = receivedMsg.getBody(byte[].class); - - if (logger.isDebugEnabled()) { - logger.debug("received msg-payload={}", new String(receivedMsgBody)); - } - - int messagesize = receivedMsgBody.length; - bytesCounter.inc(messagesize); - messagesizeHistogram.update(messagesize); - } - } catch (JMSException e) { - e.printStackTrace(); - throw new RuntimeException("Failed to acknowledge the received JMS message."); - } - } -} diff --git a/driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsMsgSendMapper.java b/driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsMsgSendMapper.java deleted file mode 100644 index fb649f013..000000000 --- a/driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsMsgSendMapper.java +++ /dev/null @@ -1,71 +0,0 @@ -/* - * Copyright (c) 2022 nosqlbench - * - * Licensed 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 io.nosqlbench.driver.jms.ops; - -import io.nosqlbench.driver.jms.JmsActivity; -import io.nosqlbench.driver.jms.util.JmsHeader; -import io.nosqlbench.driver.jms.util.JmsHeaderLongFunc; - -import javax.jms.Destination; -import java.util.Map; -import java.util.function.LongFunction; - -/** - * This maps a set of specifier functions to a pulsar operation. The pulsar operation contains - * enough state to define a pulsar operation such that it can be executed, measured, and possibly - * retried if needed. - * - * This function doesn't act *as* the operation. It merely maps the construction logic into - * a simple functional type, given the component functions. - * - * For additional parameterization, the command template is also provided. - */ -public class JmsMsgSendMapper extends JmsOpMapper { - private final JmsHeaderLongFunc jmsHeaderLongFunc; - private final Map jmsMsgProperties; - private final LongFunction msgBodyFunc; - - public JmsMsgSendMapper(JmsActivity jmsActivity, - LongFunction asyncApiFunc, - LongFunction jmsDestinationFunc, - JmsHeaderLongFunc jmsHeaderLongFunc, - Map jmsMsgProperties, - LongFunction msgBodyFunc) { - super(jmsActivity, asyncApiFunc, jmsDestinationFunc); - - this.jmsHeaderLongFunc = jmsHeaderLongFunc; - this.jmsMsgProperties = jmsMsgProperties; - this.msgBodyFunc = msgBodyFunc; - } - - @Override - public JmsOp apply(long value) { - boolean asyncApi = asyncApiFunc.apply(value); - Destination jmsDestination = jmsDestinationFunc.apply(value); - JmsHeader jmsHeader = (JmsHeader)jmsHeaderLongFunc.apply(value); - String msgBody = msgBodyFunc.apply(value); - - return new JmsMsgSendOp( - jmsActivity, - asyncApi, - jmsDestination, - jmsHeader, - jmsMsgProperties, - msgBody - ); - } -} diff --git a/driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsMsgSendOp.java b/driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsMsgSendOp.java deleted file mode 100644 index 2f432502c..000000000 --- a/driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsMsgSendOp.java +++ /dev/null @@ -1,139 +0,0 @@ -/* - * Copyright (c) 2022 nosqlbench - * - * Licensed 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 io.nosqlbench.driver.jms.ops; - -import com.codahale.metrics.Counter; -import com.codahale.metrics.Histogram; -import io.nosqlbench.driver.jms.JmsActivity; -import io.nosqlbench.driver.jms.util.JmsHeader; -import org.apache.logging.log4j.LogManager; -import org.apache.logging.log4j.Logger; - -import javax.jms.*; -import java.nio.charset.StandardCharsets; -import java.util.Iterator; -import java.util.Map; - -public class JmsMsgSendOp extends JmsTimeTrackOp { - - private final static Logger logger = LogManager.getLogger(JmsMsgSendOp.class); - - private final JmsActivity jmsActivity; - private final boolean asyncJmsOp; - private final Destination jmsDestination; - private final JmsHeader jmsHeader; - private final Map jmsMsgProperties; - - private final JMSContext jmsContext; - private final JMSProducer jmsProducer; - private final String msgBody; - - private final Counter bytesCounter; - private final Histogram messagesizeHistogram; - - public JmsMsgSendOp(JmsActivity jmsActivity, - boolean asyncJmsOp, - Destination jmsDestination, - JmsHeader jmsHeader, - Map jmsMsgProperties, - String msgBody) { - this.jmsActivity = jmsActivity; - this.asyncJmsOp = asyncJmsOp; - this.jmsDestination = jmsDestination; - - this.jmsHeader = jmsHeader; - this.jmsMsgProperties = jmsMsgProperties; - this.msgBody = msgBody; - - if (!jmsHeader.isValidHeader()) { - throw new RuntimeException(jmsHeader.getInvalidJmsHeaderMsgText()); - } - - if ((msgBody == null) || msgBody.isEmpty()) { - throw new RuntimeException("JMS message body can't be empty!"); - } - - this.jmsContext = jmsActivity.getJmsContext(); - this.jmsProducer = createJmsProducer(); - - this.bytesCounter = jmsActivity.getBytesCounter(); - this.messagesizeHistogram = jmsActivity.getMessagesizeHistogram(); - } - - private JMSProducer createJmsProducer() { - JMSProducer jmsProducer = this.jmsContext.createProducer(); - - jmsProducer.setDeliveryMode(this.jmsHeader.getDeliveryMode()); - jmsProducer.setPriority(this.jmsHeader.getMsgPriority()); - jmsProducer.setDeliveryDelay(this.jmsHeader.getMsgDeliveryDelay()); - jmsProducer.setDisableMessageTimestamp(this.jmsHeader.isDisableMsgTimestamp()); - jmsProducer.setDisableMessageID(this.jmsHeader.isDisableMsgId()); - - if (this.asyncJmsOp) { - jmsProducer.setAsync(new CompletionListener() { - @Override - public void onCompletion(Message msg) { - try { - byte[] msgBody = msg.getBody(byte[].class); - if (logger.isTraceEnabled()) { - logger.trace("Async message send success - message body: " + new String(msgBody)); - } - } - catch (JMSException jmsException) { - jmsException.printStackTrace(); - logger.warn("Unexpected error when parsing message body: " + jmsException.getMessage()); - } - } - - @Override - public void onException(Message msg, Exception e) { - try { - byte[] msgBody = msg.getBody(byte[].class); - if (logger.isTraceEnabled()) { - logger.trace("Async message send failure - message body: " + new String(msgBody)); - } - } - catch (JMSException jmsException) { - jmsException.printStackTrace(); - logger.warn("Unexpected error when parsing message body: " + jmsException.getMessage()); - } - } - }); - } - - for (Map.Entry entry : jmsMsgProperties.entrySet()) { - jmsProducer.setProperty(entry.getKey(), entry.getValue()); - } - - return jmsProducer; - } - - @Override - public void run() { - try { - byte[] msgBytes = msgBody.getBytes(StandardCharsets.UTF_8); - int messageSize = msgBytes.length; - jmsProducer.send(jmsDestination, msgBytes); - - messagesizeHistogram.update(messageSize); - bytesCounter.inc(messageSize); - } - catch (Exception ex) { - logger.error("Failed to send JMS message - " + msgBody); - } - } -} diff --git a/driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsOp.java b/driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsOp.java deleted file mode 100644 index b8f227ffd..000000000 --- a/driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsOp.java +++ /dev/null @@ -1,29 +0,0 @@ -/* - * Copyright (c) 2022 nosqlbench - * - * Licensed 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 io.nosqlbench.driver.jms.ops; - -/** - * Base type of all Pulsar Operations including Producers and Consumers. - */ -public interface JmsOp { - - /** - * Execute the operation, invoke the timeTracker when the operation ended. - * The timeTracker can be invoked in a separate thread, it is only used for metrics. - */ - void run(Runnable timeTracker); -} diff --git a/driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsOpMapper.java b/driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsOpMapper.java deleted file mode 100644 index b52385cdd..000000000 --- a/driver-jms/src/main/java/io/nosqlbench/driver/jms/ops/JmsOpMapper.java +++ /dev/null @@ -1,39 +0,0 @@ -/* - * Copyright (c) 2022 nosqlbench - * - * Licensed 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 io.nosqlbench.driver.jms.ops; - -import io.nosqlbench.driver.jms.JmsActivity; -import io.nosqlbench.driver.jms.util.JmsHeaderLongFunc; - -import javax.jms.Destination; -import java.util.Map; -import java.util.function.LongFunction; - -public abstract class JmsOpMapper implements LongFunction { - protected final JmsActivity jmsActivity; - protected final LongFunction asyncApiFunc; - protected final LongFunction jmsDestinationFunc; - - public JmsOpMapper(JmsActivity jmsActivity, - LongFunction asyncApiFunc, - LongFunction jmsDestinationFunc) - { - this.jmsActivity = jmsActivity; - this.asyncApiFunc = asyncApiFunc; - this.jmsDestinationFunc = jmsDestinationFunc; - } -} diff --git a/driver-jms/src/main/java/io/nosqlbench/driver/jms/util/JmsHeader.java b/driver-jms/src/main/java/io/nosqlbench/driver/jms/util/JmsHeader.java deleted file mode 100644 index e7d889d8a..000000000 --- a/driver-jms/src/main/java/io/nosqlbench/driver/jms/util/JmsHeader.java +++ /dev/null @@ -1,82 +0,0 @@ -/* - * Copyright (c) 2022 nosqlbench - * - * Licensed 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 io.nosqlbench.driver.jms.util; - -import lombok.AllArgsConstructor; -import lombok.Getter; -import lombok.Setter; -import lombok.ToString; -import org.apache.commons.lang.StringUtils; - -import javax.jms.DeliveryMode; - -@Setter -@Getter -@AllArgsConstructor -@ToString -public class JmsHeader { - private int deliveryMode; - private int msgPriority; - private long msgTtl; - private long msgDeliveryDelay; - private boolean disableMsgTimestamp; - private boolean disableMsgId; - - public boolean isValidDeliveryMode() { - return (deliveryMode == DeliveryMode.NON_PERSISTENT) || (deliveryMode == DeliveryMode.PERSISTENT); - } - - public boolean isValidPriority() { - return (msgPriority >= 0) && (msgPriority <= 9); - } - - public boolean isValidTtl() { - return msgTtl >= 0; - } - - public boolean isValidDeliveryDelay() { - return msgTtl >= 0; - } - - public boolean isValidHeader() { - return isValidDeliveryMode() - && isValidPriority() - && isValidTtl() - && isValidDeliveryDelay(); - } - - public String getInvalidJmsHeaderMsgText() { - StringBuilder sb = new StringBuilder(); - - if (!isValidDeliveryMode()) - sb.append("delivery mode - " + deliveryMode + "; "); - if (!isValidPriority()) - sb.append("message priority - " + msgPriority + "; "); - if (!isValidTtl()) - sb.append("message TTL - " + msgTtl + "; "); - if (!isValidDeliveryDelay()) - sb.append("message delivery delay - " + msgDeliveryDelay + "; "); - - String invalidMsgText = sb.toString(); - if (StringUtils.length(invalidMsgText) > 0) - invalidMsgText = StringUtils.substringBeforeLast(invalidMsgText, ";"); - else - invalidMsgText = "none"; - - return "Invalid JMS header values: " + invalidMsgText; - } -} diff --git a/driver-jms/src/main/java/io/nosqlbench/driver/jms/util/JmsHeaderLongFunc.java b/driver-jms/src/main/java/io/nosqlbench/driver/jms/util/JmsHeaderLongFunc.java deleted file mode 100644 index 75616091a..000000000 --- a/driver-jms/src/main/java/io/nosqlbench/driver/jms/util/JmsHeaderLongFunc.java +++ /dev/null @@ -1,47 +0,0 @@ -/* - * Copyright (c) 2022 nosqlbench - * - * Licensed 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 io.nosqlbench.driver.jms.util; - -import lombok.*; - -import javax.jms.DeliveryMode; -import javax.jms.Message; -import java.util.function.LongFunction; - -@Setter -@Getter -@NoArgsConstructor -public class JmsHeaderLongFunc implements LongFunction { - private LongFunction deliveryModeFunc; - private LongFunction msgPriorityFunc; - private LongFunction msgTtlFunc; - private LongFunction msgDeliveryDelayFunc; - private LongFunction disableMsgTimestampFunc; - private LongFunction disableMsgIdFunc; - - @Override - public Object apply(long value) { - return new JmsHeader( - (deliveryModeFunc != null) ? deliveryModeFunc.apply(value) : DeliveryMode.PERSISTENT, - (msgPriorityFunc != null) ? msgPriorityFunc.apply(value) : Message.DEFAULT_PRIORITY, - (msgTtlFunc != null) ? msgTtlFunc.apply(value) : Message.DEFAULT_TIME_TO_LIVE, - (msgTtlFunc != null) ? msgTtlFunc.apply(value) : Message.DEFAULT_DELIVERY_DELAY, - (disableMsgTimestampFunc != null) ? disableMsgTimestampFunc.apply(value) : false, - (disableMsgIdFunc != null) ? disableMsgIdFunc.apply(value) : false - ); - } -} diff --git a/driver-jms/src/main/java/io/nosqlbench/driver/jms/util/JmsUtil.java b/driver-jms/src/main/java/io/nosqlbench/driver/jms/util/JmsUtil.java deleted file mode 100644 index 278cb189b..000000000 --- a/driver-jms/src/main/java/io/nosqlbench/driver/jms/util/JmsUtil.java +++ /dev/null @@ -1,120 +0,0 @@ -/* - * Copyright (c) 2022 nosqlbench - * - * Licensed 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 io.nosqlbench.driver.jms.util; - -import org.apache.logging.log4j.LogManager; -import org.apache.logging.log4j.Logger; - -import java.util.Arrays; - -public class JmsUtil { - - private final static Logger logger = LogManager.getLogger(JmsUtil.class); - - // Supported JMS provider type - public enum JMS_PROVIDER_TYPES { - PULSAR("pulsar"); - - public final String label; - JMS_PROVIDER_TYPES(String label) { - this.label = label; - } - } - public static boolean isValidJmsProviderType(String type) { - return Arrays.stream(JMS_PROVIDER_TYPES.values()).anyMatch(t -> t.label.equals(type)); - } - - ///// - // NB command line parameters - // - JMS provider type - public final static String JMS_PROVIDER_TYPE_KEY_STR = "provider_type"; - - /// Only applicable when the provider is "Pulsar" - // - Pulsar configuration properties file - public final static String JMS_PULSAR_PROVIDER_CFG_FILE_KEY_STR = "pulsar_cfg_file"; - public final static String JMS_PULSAR_PROVIDER_DFT_CFG_FILE_NAME = "pulsar_config.properties"; - // - Pulsar web url - public final static String JMS_PULSAR_PROVIDER_WEB_URL_KEY_STR = "web_url"; - // - Pulsar service url - public final static String JMS_PULSAR_PROVIDER_SVC_URL_KEY_STR = "service_url"; - - - public final static String ASYNC_API_KEY_STR = "async_api"; - public final static String JMS_DESTINATION_TYPE_KEY_STR = "jms_desitation_type"; - - ///// JMS Producer - // Supported JMS provider type - public enum JMS_MSG_HEADER_KEYS { - DELIVERY_MODE("jms_producer_header_msg_delivery_mode"), - PRIORITY("jms_producer_header_msg_priority"), - TTL("jms_producer_header_msg_ttl"), - DELIVERY_DELAY("jms_producer_header_msg_delivery_delay"), - DISABLE_TIMESTAMP("jms_producer_header_disable_msg_timestamp"), - DISABLE_ID("jms_producer_header_disable_msg_id"); - - public final String label; - JMS_MSG_HEADER_KEYS(String label) { - this.label = label; - } - } - public static boolean isValidJmsHeaderKey(String type) { - return Arrays.stream(JMS_MSG_HEADER_KEYS.values()).anyMatch(t -> t.label.equals(type)); - } - public final static String JMS_PRODUCER_MSG_PROPERTY_KEY_STR = "jms_producer_msg_properties"; - public final static String JMS_PRODUCER_MSG_BODY_KEY_STR = "msg_body"; - - ///// JMS Consumer - public final static String JMS_CONSUMER_DURABLE_KEY_STR = "jms_consumer_msg_durable"; - public final static String JMS_CONSUMER_SHARED_KEY_STR = "jms_consumer_msg_shared"; - public final static String JMS_CONSUMER_MSG_SUBSCRIPTIOn_KEY_STR = "jms_consumer_subscription"; - public final static String JMS_CONSUMER_MSG_READ_SELECTOR_KEY_STR = "jms_consumer_msg_read_selector"; - public final static String JMS_CONSUMER_MSG_NOLOCAL_KEY_STR = "jms_consumer_msg_nolocal"; - public final static String JMS_CONSUMER_READ_TIMEOUT_KEY_STR = "jms_consumer_msg_read_timeout"; - - - // Only applicable to Pulsar JMS provider - public final static String PULSAR_JMS_TOPIC_URI_KEY_STR = "pulsar_topic_uri"; - - // Supported message operation types - public enum OP_TYPES { - MSG_SEND("msg_send"), - MSG_READ("msg_read"); - - public final String label; - OP_TYPES(String label) { - this.label = label; - } - } - public static boolean isValidClientType(String type) { - return Arrays.stream(OP_TYPES.values()).anyMatch(t -> t.label.equals(type)); - } - - // JMS Destination Types - public enum JMS_DESTINATION_TYPES { - QUEUE("queue"), - TOPIC("topic"); - - public final String label; - JMS_DESTINATION_TYPES(String label) { - this.label = label; - } - } - public static boolean isValidJmsDestinationType(String type) { - return Arrays.stream(JMS_DESTINATION_TYPES.values()).anyMatch(t -> t.label.equals(type)); - } -} - diff --git a/driver-jms/src/main/java/io/nosqlbench/driver/jms/util/PulsarConfig.java b/driver-jms/src/main/java/io/nosqlbench/driver/jms/util/PulsarConfig.java deleted file mode 100644 index 5a015cddf..000000000 --- a/driver-jms/src/main/java/io/nosqlbench/driver/jms/util/PulsarConfig.java +++ /dev/null @@ -1,115 +0,0 @@ -/* - * Copyright (c) 2022 nosqlbench - * - * Licensed 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 io.nosqlbench.driver.jms.util; - -import org.apache.commons.configuration2.Configuration; -import org.apache.commons.configuration2.FileBasedConfiguration; -import org.apache.commons.configuration2.PropertiesConfiguration; -import org.apache.commons.configuration2.builder.FileBasedConfigurationBuilder; -import org.apache.commons.configuration2.builder.fluent.Parameters; -import org.apache.commons.configuration2.ex.ConfigurationException; -import org.apache.commons.lang3.StringUtils; -import org.apache.logging.log4j.LogManager; -import org.apache.logging.log4j.Logger; - -import java.io.File; -import java.io.IOException; -import java.util.HashMap; -import java.util.Iterator; -import java.util.Map; - -public class PulsarConfig { - private final static Logger logger = LogManager.getLogger(PulsarConfig.class); - - public static final String SCHEMA_CONF_PREFIX = "schema"; - public static final String CLIENT_CONF_PREFIX = "client"; - public static final String PRODUCER_CONF_PREFIX = "producer"; - public static final String CONSUMER_CONF_PREFIX = "consumer"; - - private final Map schemaConfMap = new HashMap<>(); - private final Map clientConfMap = new HashMap<>(); - private final Map producerConfMap = new HashMap<>(); - private final Map consumerConfMap = new HashMap<>(); - - public PulsarConfig(String fileName) { - File file = new File(fileName); - - try { - String canonicalFilePath = file.getCanonicalPath(); - - Parameters params = new Parameters(); - - FileBasedConfigurationBuilder builder = - new FileBasedConfigurationBuilder(PropertiesConfiguration.class) - .configure(params.properties() - .setFileName(fileName)); - - Configuration config = builder.getConfiguration(); - - // Get schema specific configuration settings - for (Iterator it = config.getKeys(SCHEMA_CONF_PREFIX); it.hasNext(); ) { - String confKey = it.next(); - String confVal = config.getProperty(confKey).toString(); - if (!StringUtils.isBlank(confVal)) - schemaConfMap.put(confKey.substring(SCHEMA_CONF_PREFIX.length() + 1), config.getProperty(confKey)); - } - - // Get client connection specific configuration settings - for (Iterator it = config.getKeys(CLIENT_CONF_PREFIX); it.hasNext(); ) { - String confKey = it.next(); - String confVal = config.getProperty(confKey).toString(); - if (!StringUtils.isBlank(confVal)) - clientConfMap.put(confKey.substring(CLIENT_CONF_PREFIX.length() + 1), config.getProperty(confKey)); - } - - // Get producer specific configuration settings - for (Iterator it = config.getKeys(PRODUCER_CONF_PREFIX); it.hasNext(); ) { - String confKey = it.next(); - String confVal = config.getProperty(confKey).toString(); - if (!StringUtils.isBlank(confVal)) - producerConfMap.put(confKey.substring(PRODUCER_CONF_PREFIX.length() + 1), config.getProperty(confKey)); - } - - // Get consumer specific configuration settings - for (Iterator it = config.getKeys(CONSUMER_CONF_PREFIX); it.hasNext(); ) { - String confKey = it.next(); - String confVal = config.getProperty(confKey).toString(); - if (!StringUtils.isBlank(confVal)) - consumerConfMap.put(confKey.substring(CONSUMER_CONF_PREFIX.length() + 1), config.getProperty(confKey)); - } - } catch (IOException ioe) { - logger.error("Can't read the specified config properties file: " + fileName); - ioe.printStackTrace(); - } catch (ConfigurationException cex) { - logger.error("Error loading configuration items from the specified config properties file: " + fileName + ":" + cex.getMessage()); - cex.printStackTrace(); - } - } - - public Map getSchemaConfMap() { - return this.schemaConfMap; - } - public Map getClientConfMap() { - return this.clientConfMap; - } - public Map getProducerConfMap() { - return this.producerConfMap; - } - public Map getConsumerConfMap() { - return this.consumerConfMap; - } -} diff --git a/driver-jms/src/main/resources/jms.md b/driver-jms/src/main/resources/jms.md deleted file mode 100644 index 07dd0c5c7..000000000 --- a/driver-jms/src/main/resources/jms.md +++ /dev/null @@ -1 +0,0 @@ -# Overview diff --git a/driver-jms/src/main/resources/pulsar_config.properties b/driver-jms/src/main/resources/pulsar_config.properties deleted file mode 100644 index f711535ac..000000000 --- a/driver-jms/src/main/resources/pulsar_config.properties +++ /dev/null @@ -1,33 +0,0 @@ -### Schema related configurations - schema.xxx -# valid types: -# - primitive type (https://pulsar.apache.org/docs/en/schema-understand/#primitive-type) -# - keyvalue (https://pulsar.apache.org/docs/en/schema-understand/#keyvalue) -# - strut (complex type) (https://pulsar.apache.org/docs/en/schema-understand/#struct) -# avro, json, protobuf -# -# NOTE: for JMS client, Pulsar "schema" is NOT supported yet -schema.type= -schema.definition= - - -### Pulsar client related configurations - client.xxx -# http://pulsar.apache.org/docs/en/client-libraries-java/#client -client.connectionTimeoutMs=5000 -#client.authPluginClassName=org.apache.pulsar.client.impl.auth.AuthenticationToken -#client.authParams= -#client.tlsAllowInsecureConnection=true -client.numIoThreads=10 -client.numListenerThreads=10 - - -### Producer related configurations (global) - producer.xxx -# http://pulsar.apache.org/docs/en/client-libraries-java/#configure-producer -producer.sendTimeoutMs= -producer.blockIfQueueFull=true -producer.maxPendingMessages=10000 -producer.batchingMaxMessages=10000 - - -### Consumer related configurations (global) - consumer.xxx -# http://pulsar.apache.org/docs/en/client-libraries-java/#configure-consumer -consumer.receiverQueueSize=2000 diff --git a/driver-jms/src/main/resources/pulsar_jms.yaml b/driver-jms/src/main/resources/pulsar_jms.yaml deleted file mode 100644 index 755f05ed6..000000000 --- a/driver-jms/src/main/resources/pulsar_jms.yaml +++ /dev/null @@ -1,89 +0,0 @@ -bindings: - payload: NumberNameToString() #AlphaNumericString(20) - tenant: Mod(10000); Div(10L); ToString(); Prefix("tnt") - namespace: Mod(10); Div(5L); ToString(); Prefix("ns") - core_topic_name: Mod(5); ToString(); Prefix("t") - -# document level parameters that apply to all Pulsar client types: -params: - ### static only - async_api: "true" - - ### Static only - # Valid values: queue (point-to-point) or topic (pub-sub) - jms_desitation_type: "topic" - - ### Static Only - # NOTE: ONLY relevant when the JMS provider is Pulsar - #pulsar_topic_uri: "persistent://{tenant}/{namespace}/{core_topic_name}" - #pulsar_topic_uri: "persistent://public/default/pt100" - #pulsar_topic_uri: "persistent://public/default/t0" - pulsar_topic_uri: "persistent://public/default/pt100_10" - #pulsar_topic_uri: "persistent://public/default/pt200_10" - #pulsar_topic_uri: "persistent://public/default/pt300_10" - #pulsar_topic_uri: "persistent://public/default/pt400_10" - -blocks: - - name: "producer-block" - tags: - phase: "jms_producer" - statements: - - name: "s1" - optype: "msg_send" - - ### JMS PRODUCER message header - ### https://docs.oracle.com/javaee/7/api/constant-values.html#javax.jms.DeliveryMode.NON_PERSISTENT - # - static or dynamic - # - Producer only - # Valid values: non-persistent(1), or persistent(2) - default - jms_producer_header_msg_delivery_mode: "2" - # Valid values: 0~9 (4 as default) - jms_producer_header_msg_priority: "4" - # Valid values: non-negative long; default 0 (never expires) - jms_producer_header_msg_ttl: "0" - # Valid values: non-negative long; default 0 (no delay) - jms_producer_header_msg_delivery_delay: "0" - # Valid values: true/false; default false (message timestamp is enabled) - jms_producer_header_disable_msg_timestamp: "false" - # Valid values: true/false; default false (message ID is enabled) - jms_producer_header_disable_msg_id: "false" - - ### JMS PRODUCER message properties - # - static only - # - Producer only - # - In format: "key1=value1;key2=value2;..." - jms_producer_msg_properties: "key1=value1;key2=value2" - - ### JMS PRODUCER message body - msg_body: "{payload}" - - - name: "consumer-block" - tags: - phase: "jms_consumer" - statements: - - name: "s1" - optype: "msg_read" - - ### JMS CONSUMER durable and shared - jms_consumer_msg_durable: "true" - jms_consumer_msg_shared: "true" - - ### JMS CONSUMER subscription name - # - only relevant for durable consumer - jms_consumer_subscription: "mysub" - - ### JMS CONSUMER subscription name - # - only relevant for unshared consumer - jms_consumer_nolocal: "false" - - ### JMS CONSUMER message read timeout - # - unit: milliseconds - # - 0 means call blocks indefinitely - # - FIXME: 0 supposes to wait indefinitly; but - # it actually behaves like no wait at all - jms_consumer_msg_read_timeout: "10000" - - ### JMS CONSUMER message selector - # - empty string means no message selector - # - https://docs.oracle.com/cd/E19798-01/821-1841/bncer/index.html - jms_consumer_msg_read_selector: "" diff --git a/nb-api/src/main/java/io/nosqlbench/api/engine/metrics/ActivityMetrics.java b/nb-api/src/main/java/io/nosqlbench/api/engine/metrics/ActivityMetrics.java index 31008d972..bff438d72 100644 --- a/nb-api/src/main/java/io/nosqlbench/api/engine/metrics/ActivityMetrics.java +++ b/nb-api/src/main/java/io/nosqlbench/api/engine/metrics/ActivityMetrics.java @@ -83,6 +83,15 @@ public class ActivityMetrics { return metric; } + private static Metric register(String fullMetricName, MetricProvider metricProvider) { + Metric metric = get().getMetrics().get(fullMetricName); + if (metric == null) { + metric = metricProvider.getMetric(); + return get().register(fullMetricName, metric); + } + return metric; + } + private static Metric register(ScriptContext context, String name, MetricProvider metricProvider) { Metric metric = get().getMetrics().get(name); if (metric == null) { diff --git a/nb/pom.xml b/nb/pom.xml index 3cbffe028..43e93653e 100644 --- a/nb/pom.xml +++ b/nb/pom.xml @@ -67,6 +67,12 @@ 4.17.22-SNAPSHOT + + io.nosqlbench + adapter-pulsar + 4.17.22-SNAPSHOT + + diff --git a/nb5/pom.xml b/nb5/pom.xml index f2fda67a0..f801d40cd 100644 --- a/nb5/pom.xml +++ b/nb5/pom.xml @@ -88,6 +88,12 @@ 4.17.31-SNAPSHOT + + io.nosqlbench + adapter-pulsar + 4.17.31-SNAPSHOT + + diff --git a/pom.xml b/pom.xml index 9bb917de6..c8dd8eeff 100644 --- a/pom.xml +++ b/pom.xml @@ -61,6 +61,7 @@ adapter-tcp adapter-dynamodb adapter-mongodb + adapter-pulsar @@ -91,7 +92,7 @@ driver-jmx driver-jdbc driver-cockroachdb - driver-pulsar +