[
https://issues.apache.org/jira/browse/FLINK-5886?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16125762#comment-16125762
]
ASF GitHub Bot commented on FLINK-5886:
---------------------------------------
Github user zohar-mizrahi commented on a diff in the pull request:
https://github.com/apache/flink/pull/3838#discussion_r132967578
--- Diff:
flink-libraries/flink-streaming-python/src/main/java/org/apache/flink/streaming/python/api/datastream/PythonObjectInputStream2.java
---
@@ -0,0 +1,68 @@
+/*
+ * 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.flink.streaming.python.api.datastream;
+
+import org.python.util.PythonObjectInputStream;
+
+import java.io.IOException;
+import java.io.InputStream;
+import java.io.ObjectStreamClass;
+
+/**
+ * A helper class to overcome the inability to set the serialVersionUID in
a python user-defined
+ * function (UDF).
+ * <p>The fact the this field is not set, results in a dynamic calculation
of this serialVersionUID,
+ * using SHA, to make sure it is a unique number. This unique number is a
64-bit hash of the
+ * class name, interface class names, methods, and fields. If a Python
class inherits from a
+ * Java class, as in the case of Python UDFs, then a proxy wrapper class
is created. Its name is
+ * constructed using the following pattern:
+ * <b>{@code
org.python.proxies.<module-name>$<UDF-class-name>$<number>}</b>. The {@code
<number>}
+ * part is increased by one in runtime, for every job submission. It
results in different serial
+ * version UID for each run for the same Python class. Therefore, it is
required to silently
+ * suppress the serial version UID mismatch check.</p>
+ */
+public class PythonObjectInputStream2 extends PythonObjectInputStream {
+
+ public PythonObjectInputStream2(InputStream in) throws IOException {
+ super(in);
+ }
+
+ protected ObjectStreamClass readClassDescriptor() throws IOException,
ClassNotFoundException {
+ ObjectStreamClass resultClassDescriptor =
super.readClassDescriptor(); // initially streams descriptor
+
+ Class<?> localClass;
+ try {
+ localClass = resolveClass(resultClassDescriptor);
+ } catch (ClassNotFoundException e) {
+ System.out.println("No local class for " +
resultClassDescriptor.getName());
--- End diff --
The purpose of this function is to try to return the class according to the
given description. If it fails, it probably means that the Jython interpreter
was not initialised yet, and as a result it is initialised. This is handled in
`org.apache.flink.streaming.python.api.functions.UtilityFunctions::smartFunctionDeserialization`.
I'm currently checking whether the `catch` here is redundant and it's a
left-over from the debugging phase. We can probably let the exception be
propagated up the call stack.
> Python API for streaming applications
> -------------------------------------
>
> Key: FLINK-5886
> URL: https://issues.apache.org/jira/browse/FLINK-5886
> Project: Flink
> Issue Type: New Feature
> Components: Python API
> Reporter: Zohar Mizrahi
> Assignee: Zohar Mizrahi
>
> A work in progress to provide python interface for Flink streaming APIs. The
> core technology is based on jython and thus imposes two limitations: a. user
> defined functions cannot use python extensions. b. the python version is 2.x
> The branch is based on Flink release 1.2.0, as can be found here:
> https://github.com/zohar-pm/flink/tree/python-streaming
> In order to test it, someone can use IntelliJ IDE. Assuming IntelliJ was
> setup properly (see:
> https://ci.apache.org/projects/flink/flink-docs-release-1.3/internals/ide_setup.html),
> one can run/debug {{org.apache.flink.python.api.PythonStreamBinderTest}},
> which in return will execute all the tests under
> {{/Users/zohar/dev/pm-flink/flink-libraries/flink-python/src/test/python/org/apache/flink/python/api/streaming}}
--
This message was sent by Atlassian JIRA
(v6.4.14#64029)