ahmedabu98 commented on code in PR #30910:
URL: https://github.com/apache/beam/pull/30910#discussion_r1568192121


##########
model/pipeline/src/main/proto/org/apache/beam/model/pipeline/v1/schema_aware_transforms.proto:
##########
@@ -0,0 +1,73 @@
+/*
+ * 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.
+ */
+
+
+// Protocol Buffers describing available Schema-Aware transforms.
+// These are transforms that can be configured using Beam Schema-compatible 
parameters.
+// Runners can override these transforms with a native implementation.
+
+syntax = "proto3";
+
+package org.apache.beam.model.pipeline.v1;
+
+option go_package = 
"github.com/apache/beam/sdks/v2/go/pkg/beam/model/pipeline_v1;pipeline_v1";
+option java_package = "org.apache.beam.model.pipeline.v1";
+option java_outer_classname = "SchemaAwareTransforms";
+
+import "org/apache/beam/model/pipeline/v1/beam_runner_api.proto";
+import "org/apache/beam/model/pipeline/v1/schema.proto";
+
+
+message StandardSchemaAwareTransforms {
+    // Payload for all of these: SchemaAwareTransformPayload
+    enum IOs {
+        ICEBERG_READ = 0 [(beam_urn) = "beam:transform:iceberg_read:v1"];
+        ICEBERG_WRITE = 1 [(beam_urn) = "beam:transform:iceberg_write:v1"];
+    }
+    enum Managed {
+        // Payload: ManagedTransformPayload
+        MANAGED = 0 [(beam_urn) = "beam:transform:managed:v1"];
+    }
+}
+
+
+// Payload for a Schema-aware PTransform.
+// This is a transform that is aware of its input and output PCollection 
schemas
+// and is configured using Beam Schema-compatible parameters.
+message SchemaAwareTransformPayload {
+    // The transform's configuration schema
+    Schema expansion_schema = 1;
+    // The configuration used to build this transform.
+    // Must be compatible with the configuration schema, and decodable via 
beam:coder:row:v1.
+    bytes expansion_payload = 2;
+}
+
+// Payload for a Managed transform
+// This can be used by runners that wish to override an underlying transform
+// with a different implementation.
+message ManagedTransformPayload {
+    // The underlying transform's URN.
+    string underlying_transform_urn = 1;
+    // The underlying transform's configuration Schema.

Review Comment:
   Thanks, nice catch. will adjust the names and descriptions. Let me know if 
this works



##########
sdks/java/core/src/main/java/org/apache/beam/sdk/util/construction/PTransformTranslation.java:
##########
@@ -276,7 +279,7 @@ static RunnerApi.PTransform toProto(
    *
    * <p>Does not register the {@code appliedPTransform} within the provided 
{@link SdkComponents}.
    */
-  static RunnerApi.PTransform toProto(
+  public static RunnerApi.PTransform toProto(

Review Comment:
   Thanks, cleaning up



-- 
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: github-unsubscr...@beam.apache.org

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

Reply via email to