Class JmsIO.Write<EventT>

java.lang.Object
org.apache.beam.sdk.transforms.PTransform<PCollection<EventT>,WriteJmsResult<EventT>>
org.apache.beam.sdk.io.jms.JmsIO.Write<EventT>
All Implemented Interfaces:
Serializable, JmsIO.ConnectionFactoryContainer<JmsIO.Write<EventT>>, HasDisplayData
Enclosing class:
JmsIO

public abstract static class JmsIO.Write<EventT> extends PTransform<PCollection<EventT>,WriteJmsResult<EventT>> implements JmsIO.ConnectionFactoryContainer<JmsIO.Write<EventT>>
A PTransform to write to a JMS queue. See JmsIO for more information on usage and configuration.
See Also:
  • Constructor Details

    • Write

      public Write()
  • Method Details

    • withConnectionFactory

      public JmsIO.Write<EventT> withConnectionFactory(javax.jms.ConnectionFactory connectionFactory)
      Specify the JMS connection factory to connect to the JMS broker.

      The ConnectionFactory object has to be serializable, if it is not consider using the withConnectionFactoryProviderFn(org.apache.beam.sdk.transforms.SerializableFunction<java.lang.Void, ? extends javax.jms.ConnectionFactory>)

      For instance:

      
       .apply(JmsIO.write().withConnectionFactory(myConnectionFactory)
      
       
      Specified by:
      withConnectionFactory in interface JmsIO.ConnectionFactoryContainer<EventT>
      Parameters:
      connectionFactory - The JMS ConnectionFactory.
      Returns:
      The corresponding JmsIO.Read.
    • withConnectionFactoryProviderFn

      public JmsIO.Write<EventT> withConnectionFactoryProviderFn(SerializableFunction<Void,? extends javax.jms.ConnectionFactory> connectionFactoryProviderFn)
      Specify a JMS connection factory provider function to connect to the JMS broker. Use this method in case your ConnectionFactory objects are themselves not serializable, but you can recreate them as needed with a SerializableFunction provider

      For instance:

       pipeline.apply(JmsIO.write().withConnectionFactoryProviderFn(() -> new MyJmsConnectionFactory());
       
      Specified by:
      withConnectionFactoryProviderFn in interface JmsIO.ConnectionFactoryContainer<EventT>
      Parameters:
      connectionFactoryProviderFn - a SerializableFunction that creates a ConnectionFactory
      Returns:
      The corresponding JmsIO.Write
    • withQueue

      public JmsIO.Write<EventT> withQueue(String queue)
      Specify the JMS queue destination name where to send messages to. The JmsIO.Write acts as a producer on the queue.

      This method is exclusive with withTopic(String). The user has to specify a destination: queue, topic, or topicNameMapper.

      For instance:

      
       .apply(JmsIO.write().withQueue("my-queue")
      
       
      Parameters:
      queue - The JMS queue name where to send messages to.
      Returns:
      The corresponding JmsIO.Read.
    • withTopic

      public JmsIO.Write<EventT> withTopic(String topic)
      Specify the JMS topic destination name where to send messages to. The JmsIO.Read acts as a publisher on the topic.

      This method is exclusive with withQueue(String). The user has to specify a destination: queue, topic, or topicNameMapper.

      For instance:

      
       .apply(JmsIO.write().withTopic("my-topic")
      
       
      Parameters:
      topic - The JMS topic name.
      Returns:
      The corresponding JmsIO.Read.
    • withUsername

      public JmsIO.Write<EventT> withUsername(String username)
      Define the username to connect to the JMS broker (authenticated).
    • withPassword

      public JmsIO.Write<EventT> withPassword(String password)
      Define the password to connect to the JMS broker (authenticated).
    • withTopicNameMapper

      public JmsIO.Write<EventT> withTopicNameMapper(SerializableFunction<EventT,String> topicNameMapper)
      Specify the JMS topic destination name where to send messages to dynamically. The JmsIO.Write acts as a publisher on the topic.

      This method is exclusive with withQueue(String) and withTopic(String). The user has to specify a SerializableFunction that takes EventT object as a parameter, and returns the topic name depending of the content of the event object.

      For example:

      
       SerializableFunction<CompanyEvent, String> topicNameMapper =
         (event ->
          String.format(
          "company/%s/employee/%s",
          event.getCompanyName(),
          event.getEmployeeId()));
       
      
       .apply(JmsIO.write().withTopicNameMapper(topicNameNapper)
       
      Parameters:
      topicNameMapper - The function returning the dynamic topic name.
      Returns:
      The corresponding JmsIO.Write.
    • withValueMapper

      public JmsIO.Write<EventT> withValueMapper(SerializableBiFunction<EventT,javax.jms.Session,javax.jms.Message> valueMapper)
      Map the EventT object to a Message.

      For instance:

      
       SerializableBiFunction<SomeEventObject, Session, Message> valueMapper = (e, s) -> {
      
             try {
               TextMessage msg = s.createTextMessage();
               msg.setText(Mapper.MAPPER.toJson(e));
               return msg;
             } catch (JMSException ex) {
               throw new JmsIOException("Error!!", ex);
             }
           };
      
       
      
       .apply(JmsIO.write().withValueMapper(valueNapper)
       
      Parameters:
      valueMapper - The function returning the Message
      Returns:
      The corresponding JmsIO.Write.
    • withRetryConfiguration

      public JmsIO.Write<EventT> withRetryConfiguration(RetryConfiguration retryConfiguration)
      Specify the JMS retry configuration. The JmsIO.Write acts as a publisher on the topic.

      Allows a retry for failed published messages, the user should specify the maximum number of retries, a duration for retrying and a maximum cumulative retries. By default, the duration for retrying used is 15s and the maximum cumulative is 1000 days RetryConfiguration

      For example:

      
       RetryConfiguration retryConfiguration = RetryConfiguration.create(5);
       
      or
      
       RetryConfiguration retryConfiguration =
         RetryConfiguration.create(5, Duration.standardSeconds(30), null);
       
      or
      
       RetryConfiguration retryConfiguration =
         RetryConfiguration.create(5, Duration.standardSeconds(30), Duration.standardDays(15));
       
      
       .apply(JmsIO.write().withPublicationRetryPolicy(publicationRetryPolicy)
       
      Parameters:
      retryConfiguration - The retry configuration that should be used in case of failed publications.
      Returns:
      The corresponding JmsIO.Write.
    • expand

      public WriteJmsResult<EventT> expand(PCollection<EventT> input)
      Description copied from class: PTransform
      Override this method to specify how this PTransform should be expanded on the given InputT.

      NOTE: This method should not be called directly. Instead apply the PTransform should be applied to the InputT using the apply method.

      Composite transforms, which are defined in terms of other transforms, should return the output of one of the composed transforms. Non-composite transforms, which do not apply any transforms internally, should return a new unbound output and register evaluators (via backend-specific registration methods).

      Specified by:
      expand in class PTransform<PCollection<EventT>,WriteJmsResult<EventT>>