wangxianghu commented on a change in pull request #4217:
URL: https://github.com/apache/hudi/pull/4217#discussion_r762426293



##########
File path: 
hudi-utilities/src/main/java/org/apache/hudi/utilities/callback/pulsar/HoodieWriteCommitPulsarCallbackConfig.java
##########
@@ -0,0 +1,101 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hudi.utilities.callback.pulsar;
+
+import org.apache.hudi.common.config.ConfigClassProperty;
+import org.apache.hudi.common.config.ConfigGroups;
+import org.apache.hudi.common.config.ConfigProperty;
+import org.apache.hudi.common.config.HoodieConfig;
+
+import static 
org.apache.hudi.config.HoodieWriteCommitCallbackConfig.CALLBACK_PREFIX;
+
+/**
+ * Pulsar write callback related config.
+ */
+@ConfigClassProperty(name = "Write commit Pulsar callback configs",
+    groupName = ConfigGroups.Names.WRITE_CLIENT,
+    description = "Controls notifications sent to Pulsar, on events happening 
to a hudi table.")
+public class HoodieWriteCommitPulsarCallbackConfig extends HoodieConfig {
+
+  public static final ConfigProperty<String> BROKER_SERVICE_URL = 
ConfigProperty
+      .key(CALLBACK_PREFIX + "Pulsar.broker.service.url")
+      .noDefaultValue()
+      .sinceVersion("2.6.0")

Review comment:
       should be the version of hudi, right ?

##########
File path: 
hudi-utilities/src/main/java/org/apache/hudi/utilities/callback/pulsar/HoodieWriteCommitPulsarCallbackConfig.java
##########
@@ -0,0 +1,101 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hudi.utilities.callback.pulsar;
+
+import org.apache.hudi.common.config.ConfigClassProperty;
+import org.apache.hudi.common.config.ConfigGroups;
+import org.apache.hudi.common.config.ConfigProperty;
+import org.apache.hudi.common.config.HoodieConfig;
+
+import static 
org.apache.hudi.config.HoodieWriteCommitCallbackConfig.CALLBACK_PREFIX;
+
+/**
+ * Pulsar write callback related config.
+ */
+@ConfigClassProperty(name = "Write commit Pulsar callback configs",
+    groupName = ConfigGroups.Names.WRITE_CLIENT,
+    description = "Controls notifications sent to Pulsar, on events happening 
to a hudi table.")
+public class HoodieWriteCommitPulsarCallbackConfig extends HoodieConfig {
+
+  public static final ConfigProperty<String> BROKER_SERVICE_URL = 
ConfigProperty
+      .key(CALLBACK_PREFIX + "Pulsar.broker.service.url")
+      .noDefaultValue()
+      .sinceVersion("2.6.0")
+      .withDocumentation("Server's url of Pulsar cluster, to be used for 
publishing commit metadata.");
+
+  public static final ConfigProperty<String> TOPIC = ConfigProperty
+      .key(CALLBACK_PREFIX + "Pulsar.topic")
+      .noDefaultValue()
+      .sinceVersion("2.6.0")

Review comment:
       same here

##########
File path: 
hudi-utilities/src/main/java/org/apache/hudi/utilities/deltastreamer/DeltaSync.java
##########
@@ -729,9 +731,16 @@ private HoodieWriteConfig getHoodieClientConfig(Schema 
schema) {
 
     HoodieWriteConfig config = builder.build();
 
-    // set default value for {@link HoodieWriteCommitKafkaCallbackConfig} if 
needed.
-    if (config.writeCommitCallbackOn() && 
HoodieWriteCommitKafkaCallback.class.getName().equals(config.getCallbackClass()))
 {
-      
HoodieWriteCommitKafkaCallbackConfig.setCallbackKafkaConfigIfNeeded(config);
+    if (config.writeCommitCallbackOn()) {
+      // set default value for {@link HoodieWriteCommitKafkaCallbackConfig} if 
needed.
+      if 
(HoodieWriteCommitKafkaCallback.class.getName().equals(config.getCallbackClass()))
 {
+        
HoodieWriteCommitKafkaCallbackConfig.setCallbackKafkaConfigIfNeeded(config);
+      }
+
+      // set default value for {@link HoodieWriteCommitPulsarCallbackConfig} 
if needed.
+      if 
(HoodieWriteCommitPulsarCallback.class.getName().equals(config.getCallbackClass()))
 {
+        
HoodieWriteCommitPulsarCallbackConfig.setCallbackPulsarConfigIfNeeded(config);
+      }

Review comment:
       How about just set the callback config if commit callback is on without 
checking what it is , WDYT?
   

##########
File path: 
hudi-utilities/src/main/java/org/apache/hudi/utilities/callback/pulsar/HoodieWriteCommitPulsarCallback.java
##########
@@ -0,0 +1,182 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hudi.utilities.callback.pulsar;
+
+import org.apache.hudi.callback.HoodieWriteCommitCallback;
+import org.apache.hudi.callback.common.HoodieWriteCommitCallbackMessage;
+import org.apache.hudi.callback.util.HoodieWriteCommitCallbackUtil;
+import org.apache.hudi.common.config.HoodieConfig;
+import org.apache.hudi.common.util.StringUtils;
+import org.apache.hudi.common.util.ValidationUtils;
+import org.apache.hudi.config.HoodieWriteConfig;
+
+import org.apache.kafka.clients.producer.KafkaProducer;
+import org.apache.log4j.LogManager;
+import org.apache.log4j.Logger;
+import org.apache.pulsar.client.api.MessageRoutingMode;
+import org.apache.pulsar.client.api.Producer;
+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.impl.PulsarClientImpl;
+import org.apache.pulsar.client.impl.conf.ClientConfigurationData;
+
+import java.io.Closeable;
+import java.io.IOException;
+import java.time.Duration;
+import java.util.concurrent.TimeUnit;
+
+import static org.apache.hudi.common.util.DateTimeUtils.parseDuration;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.BROKER_SERVICE_URL;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.CONNECTION_TIMEOUT;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.KEEPALIVE_INTERVAL;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.OPERATION_TIMEOUT;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.PRODUCER_BLOCK_QUEUE_FULL;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.PRODUCER_PENDING_QUEUE_SIZE;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.PRODUCER_PENDING_SIZE;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.PRODUCER_ROUTE_MODE;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.PRODUCER_SEND_TIMEOUT;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.REQUEST_TIMEOUT;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.TOPIC;
+
+/**
+ * Kafka implementation of {@link HoodieWriteCommitCallback}.

Review comment:
       Pulsar implementation

##########
File path: 
hudi-utilities/src/main/java/org/apache/hudi/utilities/callback/pulsar/HoodieWriteCommitPulsarCallback.java
##########
@@ -0,0 +1,182 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hudi.utilities.callback.pulsar;
+
+import org.apache.hudi.callback.HoodieWriteCommitCallback;
+import org.apache.hudi.callback.common.HoodieWriteCommitCallbackMessage;
+import org.apache.hudi.callback.util.HoodieWriteCommitCallbackUtil;
+import org.apache.hudi.common.config.HoodieConfig;
+import org.apache.hudi.common.util.StringUtils;
+import org.apache.hudi.common.util.ValidationUtils;
+import org.apache.hudi.config.HoodieWriteConfig;
+
+import org.apache.kafka.clients.producer.KafkaProducer;
+import org.apache.log4j.LogManager;
+import org.apache.log4j.Logger;
+import org.apache.pulsar.client.api.MessageRoutingMode;
+import org.apache.pulsar.client.api.Producer;
+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.impl.PulsarClientImpl;
+import org.apache.pulsar.client.impl.conf.ClientConfigurationData;
+
+import java.io.Closeable;
+import java.io.IOException;
+import java.time.Duration;
+import java.util.concurrent.TimeUnit;
+
+import static org.apache.hudi.common.util.DateTimeUtils.parseDuration;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.BROKER_SERVICE_URL;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.CONNECTION_TIMEOUT;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.KEEPALIVE_INTERVAL;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.OPERATION_TIMEOUT;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.PRODUCER_BLOCK_QUEUE_FULL;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.PRODUCER_PENDING_QUEUE_SIZE;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.PRODUCER_PENDING_SIZE;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.PRODUCER_ROUTE_MODE;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.PRODUCER_SEND_TIMEOUT;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.REQUEST_TIMEOUT;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.TOPIC;
+
+/**
+ * Kafka implementation of {@link HoodieWriteCommitCallback}.
+ */
+public class HoodieWriteCommitPulsarCallback implements 
HoodieWriteCommitCallback, Closeable {
+
+  private static final Logger LOG = 
LogManager.getLogger(HoodieWriteCommitPulsarCallback.class);
+
+  private final String serviceUrl;
+  private final String topic;
+
+  /**
+   * The pulsar client.
+   */
+  private final transient PulsarClient client;
+
+  /**
+   * The pulsar producer.
+   */
+  private final transient Producer<String> producer;
+
+  public HoodieWriteCommitPulsarCallback(HoodieWriteConfig config) throws 
PulsarClientException {
+    this.serviceUrl = config.getString(BROKER_SERVICE_URL);
+    this.topic = config.getString(TOPIC);
+    this.client = createClient(config);
+    this.producer = createProducer(config);
+    validatePulsarConfig();
+  }
+
+  @Override
+  public void call(HoodieWriteCommitCallbackMessage callbackMessage) {
+    String callbackMsg = 
HoodieWriteCommitCallbackUtil.convertToJsonString(callbackMessage);
+    try {
+      producer.send(callbackMsg);
+      LOG.info("Send callback message succeed");
+    } catch (Exception e) {
+      LOG.error("Send kafka callback msg failed : ", e);

Review comment:
       kafka -> pulsar

##########
File path: 
hudi-utilities/src/main/java/org/apache/hudi/utilities/callback/pulsar/HoodieWriteCommitPulsarCallback.java
##########
@@ -0,0 +1,182 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hudi.utilities.callback.pulsar;
+
+import org.apache.hudi.callback.HoodieWriteCommitCallback;
+import org.apache.hudi.callback.common.HoodieWriteCommitCallbackMessage;
+import org.apache.hudi.callback.util.HoodieWriteCommitCallbackUtil;
+import org.apache.hudi.common.config.HoodieConfig;
+import org.apache.hudi.common.util.StringUtils;
+import org.apache.hudi.common.util.ValidationUtils;
+import org.apache.hudi.config.HoodieWriteConfig;
+
+import org.apache.kafka.clients.producer.KafkaProducer;
+import org.apache.log4j.LogManager;
+import org.apache.log4j.Logger;
+import org.apache.pulsar.client.api.MessageRoutingMode;
+import org.apache.pulsar.client.api.Producer;
+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.impl.PulsarClientImpl;
+import org.apache.pulsar.client.impl.conf.ClientConfigurationData;
+
+import java.io.Closeable;
+import java.io.IOException;
+import java.time.Duration;
+import java.util.concurrent.TimeUnit;
+
+import static org.apache.hudi.common.util.DateTimeUtils.parseDuration;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.BROKER_SERVICE_URL;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.CONNECTION_TIMEOUT;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.KEEPALIVE_INTERVAL;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.OPERATION_TIMEOUT;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.PRODUCER_BLOCK_QUEUE_FULL;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.PRODUCER_PENDING_QUEUE_SIZE;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.PRODUCER_PENDING_SIZE;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.PRODUCER_ROUTE_MODE;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.PRODUCER_SEND_TIMEOUT;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.REQUEST_TIMEOUT;
+import static 
org.apache.hudi.utilities.callback.pulsar.HoodieWriteCommitPulsarCallbackConfig.TOPIC;
+
+/**
+ * Kafka implementation of {@link HoodieWriteCommitCallback}.
+ */
+public class HoodieWriteCommitPulsarCallback implements 
HoodieWriteCommitCallback, Closeable {
+
+  private static final Logger LOG = 
LogManager.getLogger(HoodieWriteCommitPulsarCallback.class);
+
+  private final String serviceUrl;
+  private final String topic;
+
+  /**
+   * The pulsar client.
+   */
+  private final transient PulsarClient client;
+
+  /**
+   * The pulsar producer.
+   */
+  private final transient Producer<String> producer;
+
+  public HoodieWriteCommitPulsarCallback(HoodieWriteConfig config) throws 
PulsarClientException {
+    this.serviceUrl = config.getString(BROKER_SERVICE_URL);
+    this.topic = config.getString(TOPIC);
+    this.client = createClient(config);
+    this.producer = createProducer(config);
+    validatePulsarConfig();
+  }
+
+  @Override
+  public void call(HoodieWriteCommitCallbackMessage callbackMessage) {
+    String callbackMsg = 
HoodieWriteCommitCallbackUtil.convertToJsonString(callbackMessage);
+    try {
+      producer.send(callbackMsg);
+      LOG.info("Send callback message succeed");
+    } catch (Exception e) {
+      LOG.error("Send kafka callback msg failed : ", e);
+    }
+  }
+
+  /**
+   * Method helps to create {@link KafkaProducer}. Here we set acks = all and 
retries = 3 by default to ensure no data
+   * loss.
+   *
+   * @param hoodieConfig Kafka configs
+   * @return A {@link KafkaProducer}
+   */
+  public Producer<String> createProducer(HoodieConfig hoodieConfig) throws 
PulsarClientException {

Review comment:
       please correct the doc




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

To unsubscribe, e-mail: [email protected]

For queries about this service, please contact Infrastructure at:
[email protected]


Reply via email to