This functionality is not yet implemented. + * + *
The Cloud Spanner {@link SpannerIO.Write} transform writes to Cloud Spanner by executing a + * collection of input row {@link Mutation Mutations}. The mutations grouped into batches for + * efficiency. + * + *
To configure the write transform, create an instance using {@link #write()} and then specify + * the destination Cloud Spanner instance ({@link Write#withInstanceId(String)} and destination + * database ({@link Write#withDatabaseId(String)}). For example: + * + *
{@code
+ * // Earlier in the pipeline, create a PCollection of Mutations to be written to Cloud Spanner.
+ * PCollection mutations = ...;
+ * // Write mutations.
+ * mutations.apply(
+ * "Write", SpannerIO.write().withInstanceId("instance").withDatabaseId("database"));
+ * }
+ */
+@Experimental(Experimental.Kind.SOURCE_SINK)
+public class SpannerIO {
+
+ @VisibleForTesting
+ static final int SPANNER_MUTATIONS_PER_COMMIT_LIMIT = 20000;
+
+ /**
+ * Creates an unitialized instance of {@link Write}. Before use, the {@link Write} must be
+ * configured with a {@link Write#withInstanceId} and {@link Write#withDatabaseId} that identify
+ * the Cloud Spanner database being written.
+ */
+ @Experimental
+ public static Write write() {
+ return new AutoValue_SpannerIO_Write.Builder().build();
+ }
+
+ /**
+ * A {@link PTransform} that writes {@link Mutation} objects to Google Cloud Spanner.
+ *
+ * @see SpannerIO
+ */
+ @Experimental(Experimental.Kind.SOURCE_SINK)
+ @AutoValue
+ public abstract static class Write extends PTransformDoes not modify this object. + */ + public Write withInstanceId(String instanceId) { + return toBuilder().setInstanceId(instanceId).build(); + } + + /** + * Returns a new {@link SpannerIO.Write} that will write to the specified Cloud Spanner + * database. + * + *
Does not modify this object.
+ */
+ public Write withDatabaseId(String databaseId) {
+ return toBuilder().setDatabaseId(databaseId).build();
+ }
+
+ @Override
+ public PDone expand(PCollection Commits are non-transactional. If a commit fails, it will be retried (up to
+ * {@link SpannerWriterFn#MAX_RETRIES} times). This means that the mutation operation should be
+ * idempotent.
+ *
+ * See Google Cloud Spanner documentation.
+ */
+ @VisibleForTesting
+ static class SpannerWriterFn extends DoFn If a commit fails, it will be retried up to {@link #MAX_RETRIES} times.
+ * If the retry limit is exceeded, the last exception from Cloud Spanner will be
+ * thrown.
+ *
+ * @throws AbortedException if the commit fails or IOException or InterruptedException if
+ * backing off between retries fails.
+ */
+ private void flushBatch() throws AbortedException, IOException, InterruptedException {
+ LOG.debug("Writing batch of {} mutations", mutations.size());
+ Sleeper sleeper = Sleeper.DEFAULT;
+ BackOff backoff = BUNDLE_WRITE_BACKOFF.backoff();
+
+ while (true) {
+ // Batch upsert rows.
+ try {
+ dbClient.writeAtLeastOnce(mutations);
+
+ // Break if the commit threw no exception.
+ break;
+ } catch (AbortedException exception) {
+ // Only log the code and message for potentially-transient errors. The entire exception
+ // will be propagated upon the last retry.
+ LOG.error("Error writing to Spanner ({}): {}", exception.getCode(),
+ exception.getMessage());
+ if (!BackOffUtils.next(sleeper, backoff)) {
+ LOG.error("Aborting after {} retries.", MAX_RETRIES);
+ throw exception;
+ }
+ }
+ }
+ LOG.debug("Successfully wrote {} mutations", mutations.size());
+ mutations.clear();
+ }
+
+ @Override
+ public void populateDisplayData(Builder builder) {
+ super.populateDisplayData(builder);
+ builder
+ .addIfNotNull(DisplayData.item("instanceId", instanceId)
+ .withLabel("Instance"))
+ .addIfNotNull(DisplayData.item("databaseId", databaseId)
+ .withLabel("Database"));
+ }
+ }
+
+ private SpannerIO() {} // Prevent construction.
+
+}
diff --git a/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/spanner/package-info.java b/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/spanner/package-info.java
new file mode 100644
index 000000000000..19e468cce041
--- /dev/null
+++ b/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/spanner/package-info.java
@@ -0,0 +1,23 @@
+/*
+ * 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.
+ */
+
+/**
+ * Provides an API for reading from and writing to
+ * Google Cloud Spanner.
+ */
+package org.apache.beam.sdk.io.gcp.spanner;
diff --git a/sdks/java/io/google-cloud-platform/src/test/java/org/apache/beam/sdk/io/gcp/GcpApiSurfaceTest.java b/sdks/java/io/google-cloud-platform/src/test/java/org/apache/beam/sdk/io/gcp/GcpApiSurfaceTest.java
index 7025004cc953..91caded1ad35 100644
--- a/sdks/java/io/google-cloud-platform/src/test/java/org/apache/beam/sdk/io/gcp/GcpApiSurfaceTest.java
+++ b/sdks/java/io/google-cloud-platform/src/test/java/org/apache/beam/sdk/io/gcp/GcpApiSurfaceTest.java
@@ -63,7 +63,10 @@ public void testGcpApiSurface() throws Exception {
Matchers.