[ 
https://issues.apache.org/jira/browse/BEAM-9894?focusedWorklogId=439895&page=com.atlassian.jira.plugin.system.issuetabpanels:worklog-tabpanel#worklog-439895
 ]

ASF GitHub Bot logged work on BEAM-9894:
----------------------------------------

                Author: ASF GitHub Bot
            Created on: 02/Jun/20 05:27
            Start Date: 02/Jun/20 05:27
    Worklog Time Spent: 10m 
      Work Description: purbanow commented on a change in pull request #11794:
URL: https://github.com/apache/beam/pull/11794#discussion_r433627687



##########
File path: 
sdks/java/io/snowflake/src/main/java/org/apache/beam/sdk/io/snowflake/SnowflakeIO.java
##########
@@ -447,6 +494,346 @@ public void populateDisplayData(DisplayData.Builder 
builder) {
     }
   }
 
+  /** Implementation of {@link #write()}. */
+  @AutoValue
+  public abstract static class Write<T> extends PTransform<PCollection<T>, 
PDone> {
+    @Nullable
+    abstract SerializableFunction<Void, DataSource> getDataSourceProviderFn();
+
+    @Nullable
+    abstract String getTable();
+
+    @Nullable
+    abstract String getQuery();
+
+    @Nullable
+    abstract Location getLocation();
+
+    @Nullable
+    abstract String getFileNameTemplate();
+
+    @Nullable
+    abstract WriteDisposition getWriteDisposition();
+
+    @Nullable
+    abstract UserDataMapper getUserDataMapper();
+
+    @Nullable
+    abstract SnowflakeService getSnowflakeService();
+
+    abstract Builder<T> toBuilder();
+
+    @AutoValue.Builder
+    abstract static class Builder<T> {
+      abstract Builder<T> setDataSourceProviderFn(
+          SerializableFunction<Void, DataSource> dataSourceProviderFn);
+
+      abstract Builder<T> setTable(String table);
+
+      abstract Builder<T> setQuery(String query);
+
+      abstract Builder<T> setLocation(Location location);
+
+      abstract Builder<T> setFileNameTemplate(String fileNameTemplate);
+
+      abstract Builder<T> setUserDataMapper(UserDataMapper userDataMapper);
+
+      abstract Builder<T> setWriteDisposition(WriteDisposition 
writeDisposition);
+
+      abstract Builder<T> setSnowflakeService(SnowflakeService 
snowflakeService);
+
+      abstract Write<T> build();
+    }
+
+    /**
+     * Setting information about Snowflake server.
+     *
+     * @param config - An instance of {@link DataSourceConfiguration}.
+     */
+    public Write<T> withDataSourceConfiguration(final DataSourceConfiguration 
config) {
+      return withDataSourceProviderFn(new 
DataSourceProviderFromDataSourceConfiguration(config));
+    }
+
+    /**
+     * Setting function that will provide {@link DataSourceConfiguration} in 
runtime.
+     *
+     * @param dataSourceProviderFn a {@link SerializableFunction}.
+     */
+    public Write<T> withDataSourceProviderFn(
+        SerializableFunction<Void, DataSource> dataSourceProviderFn) {
+      return toBuilder().setDataSourceProviderFn(dataSourceProviderFn).build();
+    }
+
+    /**
+     * A table name to be written in Snowflake.
+     *
+     * @param table - String with the name of the table.
+     */
+    public Write<T> to(String table) {
+      return toBuilder().setTable(table).build();
+    }
+
+    /**
+     * A query to be executed in Snowflake.
+     *
+     * @param query - String with query.
+     */
+    public Write<T> withQueryTransformation(String query) {
+      return toBuilder().setQuery(query).build();
+    }
+
+    /**
+     * A location object which contains connection config between Snowflake 
and GCP.
+     *
+     * @param location - an instance of {@link Location}.
+     */
+    public Write<T> via(Location location) {
+      return toBuilder().setLocation(location).build();
+    }
+
+    /**
+     * A template name for files saved to GCP.
+     *
+     * @param fileNameTemplate - String with template name for files.
+     */
+    public Write<T> withFileNameTemplate(String fileNameTemplate) {
+      return toBuilder().setFileNameTemplate(fileNameTemplate).build();
+    }
+
+    /**
+     * User-defined function mapping user data into CSV lines.
+     *
+     * @param userDataMapper - an instance of {@link UserDataMapper}.
+     */
+    public Write<T> withUserDataMapper(UserDataMapper userDataMapper) {
+      return toBuilder().setUserDataMapper(userDataMapper).build();
+    }
+
+    /**
+     * A disposition to be used during writing to table phase.
+     *
+     * @param writeDisposition - an instance of {@link WriteDisposition}.
+     */
+    public Write<T> withWriteDisposition(WriteDisposition writeDisposition) {
+      return toBuilder().setWriteDisposition(writeDisposition).build();
+    }
+
+    /**
+     * A snowflake service which is supposed to be used. Note: Currently we 
have {@link
+     * SnowflakeServiceImpl} with corresponding {@link 
FakeSnowflakeServiceImpl} used for testing.
+     *
+     * @param snowflakeService - an instance of {@link SnowflakeService}.
+     */
+    public Write<T> withSnowflakeService(SnowflakeService snowflakeService) {
+      return toBuilder().setSnowflakeService(snowflakeService).build();
+    }
+
+    @Override
+    public PDone expand(PCollection<T> input) {
+      Location loc = getLocation();
+      checkArguments(loc);
+
+      String stagingBucketDir = String.format("%s/%s/", 
loc.getStagingBucketName(), WRITE_TMP_PATH);
+
+      PCollection out = write(input, stagingBucketDir);
+      out.setCoder(StringUtf8Coder.of());
+
+      return PDone.in(out.getPipeline());
+    }
+
+    private void checkArguments(Location loc) {
+      checkArgument(loc != null, "via() is required");
+      checkArgument(
+          loc.getStorageIntegrationName() != null,
+          "location with storageIntegrationName is required");
+      checkArgument(
+          loc.getStagingBucketName() != null, "location with stagingBucketName 
is required");
+
+      checkArgument(getUserDataMapper() != null, "withUserDataMapper() is 
required");
+
+      checkArgument(
+          (getDataSourceProviderFn() != null),
+          "withDataSourceConfiguration() or withDataSourceProviderFn() is 
required");
+
+      checkArgument(getTable() != null, "withTable() is required");
+    }
+
+    private PCollection write(PCollection input, String stagingBucketDir) {
+      SnowflakeService snowflakeService =
+          getSnowflakeService() != null ? getSnowflakeService() : new 
SnowflakeServiceImpl();
+
+      PCollection files = writeFiles(input, stagingBucketDir);
+
+      files =
+          (PCollection)
+              files.apply("Create list of files to copy", Combine.globally(new 
Concatenate()));
+
+      return (PCollection)
+          files.apply("Copy files to table", copyToTable(snowflakeService, 
stagingBucketDir));
+    }
+
+    private PCollection writeFiles(PCollection<T> input, String 
stagingBucketDir) {
+      class Parse extends DoFn<KV<T, String>, String> {
+        @ProcessElement
+        public void processElement(ProcessContext c) {
+          c.output(c.element().getValue());
+        }
+      }
+
+      PCollection mappedUserData =
+          input
+              .apply(
+                  "Map user data to Objects array",
+                  ParDo.of(new 
MapUserDataObjectsArrayFn<T>(getUserDataMapper())))
+              .apply("Map Objects array to CSV lines", ParDo.of(new 
MapObjectsArrayToCsvFn()))
+              .setCoder(StringUtf8Coder.of());
+
+      WriteFilesResult filesResult =
+          (WriteFilesResult)
+              mappedUserData.apply(
+                  "Write files to specified location",
+                  FileIO.write()
+                      .via((FileIO.Sink) new CSVSink())
+                      .to(stagingBucketDir)
+                      .withPrefix(getFileNameTemplate())
+                      .withSuffix(".csv")
+                      .withPrefix(UUID.randomUUID().toString().subSequence(0, 
8).toString())
+                      .withCompression(Compression.GZIP));
+
+      return (PCollection)
+          filesResult
+              .getPerDestinationOutputFilenames()
+              .apply("Parse KV filenames to Strings", ParDo.of(new Parse()));
+    }
+
+    private ParDo.SingleOutput<Object, Object> copyToTable(
+        SnowflakeService snowflakeService, String stagingBucketDir) {
+      return ParDo.of(
+          new CopyToTableFn<>(
+              getDataSourceProviderFn(),
+              getTable(),
+              getQuery(),
+              stagingBucketDir,
+              getLocation(),
+              getWriteDisposition(),
+              snowflakeService));
+    }
+  }
+
+  public static class Concatenate extends Combine.CombineFn<String, 
List<String>, List<String>> {
+    @Override
+    public List<String> createAccumulator() {
+      return new ArrayList<>();
+    }
+
+    @Override
+    public List<String> addInput(List<String> mutableAccumulator, String 
input) {
+      mutableAccumulator.add(String.format("'%s'", input));
+      return mutableAccumulator;
+    }
+
+    @Override
+    public List<String> mergeAccumulators(Iterable<List<String>> accumulators) 
{
+      List<String> result = createAccumulator();
+      for (List<String> accumulator : accumulators) {
+        result.addAll(accumulator);
+      }
+      return result;
+    }
+
+    @Override
+    public List<String> extractOutput(List<String> accumulator) {
+      return accumulator;
+    }
+  }
+
+  private static class MapUserDataObjectsArrayFn<T> extends DoFn<T, Object[]> {
+    private final UserDataMapper<T> csvMapper;
+
+    public MapUserDataObjectsArrayFn(UserDataMapper<T> csvMapper) {
+      this.csvMapper = csvMapper;
+    }
+
+    @ProcessElement
+    public void processElement(ProcessContext context) throws Exception {
+      context.output(csvMapper.mapRow(context.element()));
+    }
+  }
+
+  /**
+   * Custom DoFn that maps {@link Object[]} into CSV line to be saved to 
Snowflake.
+   *
+   * <p>Adds Snowflake-specific quotations around strings.
+   */
+  private static class MapObjectsArrayToCsvFn extends DoFn<Object[], String> {
+
+    @ProcessElement
+    public void processElement(ProcessContext context) {
+      List<Object> csvItems = new ArrayList<>();
+      for (Object o : context.element()) {
+        if (o instanceof String) {
+          String field = (String) o;
+          field = field.replace("'", "''");
+          field = quoteField(field);
+
+          csvItems.add(field);
+        } else {
+          csvItems.add(o);

Review comment:
       Based on  `If the data contains single or double quotes, then those 
quotes must be escaped.` proper is  `'I can\'t believe it'`




----------------------------------------------------------------
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.

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org


Issue Time Tracking
-------------------

    Worklog Id:     (was: 439895)
    Time Spent: 6h  (was: 5h 50m)

> Add batch SnowflakeIO.Write to Java SDK
> ---------------------------------------
>
>                 Key: BEAM-9894
>                 URL: https://issues.apache.org/jira/browse/BEAM-9894
>             Project: Beam
>          Issue Type: New Feature
>          Components: io-ideas
>            Reporter: Dariusz Aniszewski
>            Assignee: Pawel Urbanowicz
>            Priority: P2
>          Time Spent: 6h
>  Remaining Estimate: 0h
>




--
This message was sent by Atlassian Jira
(v8.3.4#803005)

Reply via email to