This is an automated email from the ASF dual-hosted git repository.

jincheng pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/flink.git


The following commit(s) were added to refs/heads/master by this push:
     new 68bba73  [FLINK-14198][python] Add "type" and "rtype" options to flink 
python API docstrings of table.py and table_environment.py
68bba73 is described below

commit 68bba73ab705b7efe9c984235dbf0c20f64c375c
Author: Wei Zhong <weizhong0...@gmail.com>
AuthorDate: Tue Dec 3 10:17:42 2019 +0800

    [FLINK-14198][python] Add "type" and "rtype" options to flink python API 
docstrings of table.py and table_environment.py
    
    This closes #10389
---
 flink-python/pyflink/table/table.py             | 235 +++++++++++++++---------
 flink-python/pyflink/table/table_environment.py | 203 ++++++++++++--------
 2 files changed, 278 insertions(+), 160 deletions(-)

diff --git a/flink-python/pyflink/table/table.py 
b/flink-python/pyflink/table/table.py
index 2feaf52..5285784 100644
--- a/flink-python/pyflink/table/table.py
+++ b/flink-python/pyflink/table/table.py
@@ -20,7 +20,6 @@ from py4j.java_gateway import get_method
 from pyflink.java_gateway import get_gateway
 from pyflink.table.table_schema import TableSchema
 
-from pyflink.table.window import GroupWindow
 from pyflink.util.utils import to_jarray
 
 __all__ = ['Table', 'GroupedTable', 'GroupWindowedTable', 'OverWindowedTable', 
'WindowGroupedTable']
@@ -29,11 +28,11 @@ __all__ = ['Table', 'GroupedTable', 'GroupWindowedTable', 
'OverWindowedTable', '
 class Table(object):
 
     """
-    A :class:`Table` is the core component of the Table API.
+    A :class:`~pyflink.table.Table` is the core component of the Table API.
     Similar to how the batch and streaming APIs have DataSet and DataStream,
-    the Table API is built around :class:`Table`.
+    the Table API is built around :class:`~pyflink.table.Table`.
 
-    Use the methods of :class:`Table` to transform data.
+    Use the methods of :class:`~pyflink.table.Table` to transform data.
 
     Example:
     ::
@@ -70,7 +69,9 @@ class Table(object):
             >>> tab.select("key, value + 'hello'")
 
         :param fields: Expression string.
-        :return: The result :class:`Table`.
+        :type fields: str
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.select(fields))
 
@@ -85,7 +86,9 @@ class Table(object):
             >>> tab.alias("a, b")
 
         :param fields: Field list expression string.
-        :return: The result :class:`Table`.
+        :type fields: str
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(get_method(self._j_table, "as")(fields))
 
@@ -100,7 +103,9 @@ class Table(object):
             >>> tab.filter("name = 'Fred'")
 
         :param predicate: Predicate expression string.
-        :return: The result :class:`Table`.
+        :type predicate: str
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.filter(predicate))
 
@@ -115,7 +120,9 @@ class Table(object):
             >>> tab.where("name = 'Fred'")
 
         :param predicate: Predicate expression string.
-        :return: The result :class:`Table`.
+        :type predicate: str
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.where(predicate))
 
@@ -130,7 +137,9 @@ class Table(object):
             >>> tab.group_by("key").select("key, value.avg")
 
         :param fields: Group keys.
-        :return: The grouped :class:`Table`.
+        :type fields: str
+        :return: The grouped table.
+        :rtype: pyflink.table.GroupedTable
         """
         return GroupedTable(self._j_table.groupBy(fields))
 
@@ -143,20 +152,21 @@ class Table(object):
 
             >>> tab.select("key, value").distinct()
 
-        :return: The result :class:`Table`.
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.distinct())
 
     def join(self, right, join_predicate=None):
         """
-        Joins two :class:`Table`. Similar to a SQL join. The fields of the two 
joined
+        Joins two :class:`~pyflink.table.Table`. Similar to a SQL join. The 
fields of the two joined
         operations must not overlap, use :func:`~pyflink.table.Table.alias` to 
rename fields if
         necessary. You can use where and select clauses after a join to 
further specify the
         behaviour of the join.
 
         .. note::
 
-            Both tables must be bound to the same :class:`TableEnvironment` .
+            Both tables must be bound to the same 
:class:`~pyflink.table.TableEnvironment` .
 
         Example:
         ::
@@ -165,8 +175,11 @@ class Table(object):
             >>> left.join(right, "a = b")
 
         :param right: Right table.
+        :type right: pyflink.table.Table
         :param join_predicate: Optional, the join predicate expression string.
-        :return: The result :class:`Table`.
+        :type join_predicate: str
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         if join_predicate is not None:
             return Table(self._j_table.join(right._j_table, join_predicate))
@@ -175,14 +188,14 @@ class Table(object):
 
     def left_outer_join(self, right, join_predicate=None):
         """
-        Joins two :class:`Table`. Similar to a SQL left outer join. The fields 
of the two joined
-        operations must not overlap, use :func:`~pyflink.table.Table.alias` to 
rename fields if
-        necessary.
+        Joins two :class:`~pyflink.table.Table`. Similar to a SQL left outer 
join. The fields of
+        the two joined operations must not overlap, use 
:func:`~pyflink.table.Table.alias` to
+        rename fields if necessary.
 
         .. note::
 
-            Both tables must be bound to the same :class:`TableEnvironment` 
and its
-            :class:`TableConfig` must have null check enabled (default).
+            Both tables must be bound to the same 
:class:`~pyflink.table.TableEnvironment` and its
+            :class:`~pyflink.table.TableConfig` must have null check enabled 
(default).
 
         Example:
         ::
@@ -191,8 +204,11 @@ class Table(object):
             >>> left.left_outer_join(right, "a = b").select("a, b, d")
 
         :param right: Right table.
+        :type right: pyflink.table.Table
         :param join_predicate: Optional, the join predicate expression string.
-        :return: The result :class:`Table`.
+        :type join_predicate: str
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         if join_predicate is None:
             return Table(self._j_table.leftOuterJoin(right._j_table))
@@ -201,14 +217,14 @@ class Table(object):
 
     def right_outer_join(self, right, join_predicate):
         """
-        Joins two :class:`Table`. Similar to a SQL right outer join. The 
fields of the two joined
-        operations must not overlap, use :func:`~pyflink.table.Table.alias` to 
rename fields if
-        necessary.
+        Joins two :class:`~pyflink.table.Table`. Similar to a SQL right outer 
join. The fields of
+        the two joined operations must not overlap, use 
:func:`~pyflink.table.Table.alias` to
+        rename fields if necessary.
 
         .. note::
 
-            Both tables must be bound to the same :class:`TableEnvironment` 
and its
-            :class:`TableConfig` must have null check enabled (default).
+            Both tables must be bound to the same 
:class:`~pyflink.table.TableEnvironment` and its
+            :class:`~pyflink.table.TableConfig` must have null check enabled 
(default).
 
         Example:
         ::
@@ -216,21 +232,24 @@ class Table(object):
             >>> left.right_outer_join(right, "a = b").select("a, b, d")
 
         :param right: Right table.
+        :type right: pyflink.table.Table
         :param join_predicate: The join predicate expression string.
-        :return: The result :class:`Table`.
+        :type join_predicate: str
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.rightOuterJoin(right._j_table, 
join_predicate))
 
     def full_outer_join(self, right, join_predicate):
         """
-        Joins two :class:`Table`. Similar to a SQL full outer join. The fields 
of the two joined
-        operations must not overlap, use :func:`~pyflink.table.Table.alias` to 
rename fields if
-        necessary.
+        Joins two :class:`~pyflink.table.Table`. Similar to a SQL full outer 
join. The fields of
+        the two joined operations must not overlap, use 
:func:`~pyflink.table.Table.alias` to
+        rename fields if necessary.
 
         .. note::
 
-            Both tables must be bound to the same :class:`TableEnvironment` 
and its
-            :class:`TableConfig` must have null check enabled (default).
+            Both tables must be bound to the same 
:class:`~pyflink.table.TableEnvironment` and its
+            :class:`~pyflink.table.TableConfig` must have null check enabled 
(default).
 
         Example:
         ::
@@ -238,8 +257,11 @@ class Table(object):
             >>> left.full_outer_join(right, "a = b").select("a, b, d")
 
         :param right: Right table.
+        :type right: pyflink.table.Table
         :param join_predicate: The join predicate expression string.
-        :return: The result :class:`Table`.
+        :type join_predicate: str
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.fullOuterJoin(right._j_table, 
join_predicate))
 
@@ -261,7 +283,7 @@ class Table(object):
                                exist.
         :type join_predicate: str
         :return: The result Table.
-        :rtype: Table
+        :rtype: pyflink.table.Table
         """
         if join_predicate is None:
             return Table(self._j_table.joinLateral(table_function_call))
@@ -287,7 +309,7 @@ class Table(object):
                                exist.
         :type join_predicate: str
         :return: The result Table.
-        :rtype: Table
+        :rtype: pyflink.table.Table
         """
         if join_predicate is None:
             return 
Table(self._j_table.leftOuterJoinLateral(table_function_call))
@@ -296,14 +318,14 @@ class Table(object):
 
     def minus(self, right):
         """
-        Minus of two :class:`Table` with duplicate records removed.
+        Minus of two :class:`~pyflink.table.Table` with duplicate records 
removed.
         Similar to a SQL EXCEPT clause. Minus returns records from the left 
table that do not
         exist in the right table. Duplicate records in the left table are 
returned
         exactly once, i.e., duplicates are removed. Both tables must have 
identical field types.
 
         .. note::
 
-            Both tables must be bound to the same :class:`TableEnvironment`.
+            Both tables must be bound to the same 
:class:`~pyflink.table.TableEnvironment`.
 
         Example:
         ::
@@ -311,13 +333,15 @@ class Table(object):
             >>> left.minus(right)
 
         :param right: Right table.
-        :return: The result :class:`Table`.
+        :type right: pyflink.table.Table
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.minus(right._j_table))
 
     def minus_all(self, right):
         """
-        Minus of two :class:`Table`. Similar to a SQL EXCEPT ALL.
+        Minus of two :class:`~pyflink.table.Table`. Similar to a SQL EXCEPT 
ALL.
         Similar to a SQL EXCEPT ALL clause. MinusAll returns the records that 
do not exist in
         the right table. A record that is present n times in the left table 
and m times
         in the right table is returned (n - m) times, i.e., as many duplicates 
as are present
@@ -325,7 +349,7 @@ class Table(object):
 
         .. note::
 
-            Both tables must be bound to the same :class:`TableEnvironment`.
+            Both tables must be bound to the same 
:class:`~pyflink.table.TableEnvironment`.
 
         Example:
         ::
@@ -333,18 +357,20 @@ class Table(object):
             >>> left.minus_all(right)
 
         :param right: Right table.
-        :return: The result :class:`Table`.
+        :type right: pyflink.table.Table
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.minusAll(right._j_table))
 
     def union(self, right):
         """
-        Unions two :class:`Table` with duplicate records removed.
+        Unions two :class:`~pyflink.table.Table` with duplicate records 
removed.
         Similar to a SQL UNION. The fields of the two union operations must 
fully overlap.
 
         .. note::
 
-            Both tables must be bound to the same :class:`TableEnvironment`.
+            Both tables must be bound to the same 
:class:`~pyflink.table.TableEnvironment`.
 
         Example:
         ::
@@ -352,18 +378,20 @@ class Table(object):
             >>> left.union(right)
 
         :param right: Right table.
-        :return: The result :class:`Table`.
+        :type right: pyflink.table.Table
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.union(right._j_table))
 
     def union_all(self, right):
         """
-        Unions two :class:`Table`. Similar to a SQL UNION ALL. The fields of 
the two union
-        operations must fully overlap.
+        Unions two :class:`~pyflink.table.Table`. Similar to a SQL UNION ALL. 
The fields of the
+        two union operations must fully overlap.
 
         .. note::
 
-            Both tables must be bound to the same :class:`TableEnvironment`.
+            Both tables must be bound to the same 
:class:`~pyflink.table.TableEnvironment`.
 
         Example:
         ::
@@ -371,20 +399,23 @@ class Table(object):
             >>> left.union_all(right)
 
         :param right: Right table.
-        :return: The result :class:`Table`.
+        :type right: pyflink.table.Table
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.unionAll(right._j_table))
 
     def intersect(self, right):
         """
-        Intersects two :class:`Table` with duplicate records removed. 
Intersect returns records
-        that exist in both tables. If a record is present in one or both 
tables more than once,
-        it is returned just once, i.e., the resulting table has no duplicate 
records. Similar to a
-        SQL INTERSECT. The fields of the two intersect operations must fully 
overlap.
+        Intersects two :class:`~pyflink.table.Table` with duplicate records 
removed. Intersect
+        returns records that exist in both tables. If a record is present in 
one or both tables
+        more than once, it is returned just once, i.e., the resulting table 
has no duplicate
+        records. Similar to a SQL INTERSECT. The fields of the two intersect 
operations must fully
+        overlap.
 
         .. note::
 
-            Both tables must be bound to the same :class:`TableEnvironment`.
+            Both tables must be bound to the same 
:class:`~pyflink.table.TableEnvironment`.
 
         Example:
         ::
@@ -392,20 +423,23 @@ class Table(object):
             >>> left.intersect(right)
 
         :param right: Right table.
-        :return: The result :class:`Table`.
+        :type right: pyflink.table.Table
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.intersect(right._j_table))
 
     def intersect_all(self, right):
         """
-        Intersects two :class:`Table`. IntersectAll returns records that exist 
in both tables.
-        If a record is present in both tables more than once, it is returned 
as many times as it
-        is present in both tables, i.e., the resulting table might have 
duplicate records. Similar
-        to an SQL INTERSECT ALL. The fields of the two intersect operations 
must fully overlap.
+        Intersects two :class:`~pyflink.table.Table`. IntersectAll returns 
records that exist in
+        both tables. If a record is present in both tables more than once, it 
is returned as many
+        times as it is present in both tables, i.e., the resulting table might 
have duplicate
+        records. Similar to an SQL INTERSECT ALL. The fields of the two 
intersect operations must
+        fully overlap.
 
         .. note::
 
-            Both tables must be bound to the same :class:`TableEnvironment`.
+            Both tables must be bound to the same 
:class:`~pyflink.table.TableEnvironment`.
 
         Example:
         ::
@@ -413,13 +447,15 @@ class Table(object):
             >>> left.intersect_all(right)
 
         :param right: Right table.
-        :return: The result :class:`Table`.
+        :type right: pyflink.table.Table
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.intersectAll(right._j_table))
 
     def order_by(self, fields):
         """
-        Sorts the given :class:`Table`. Similar to SQL ORDER BY.
+        Sorts the given :class:`~pyflink.table.Table`. Similar to SQL ORDER BY.
         The resulting Table is sorted globally sorted across all parallel 
partitions.
 
         Example:
@@ -427,8 +463,10 @@ class Table(object):
 
             >>> tab.order_by("name.desc")
 
-        :param fields: Order fields expression string,
-        :return: The result :class:`Table`.
+        :param fields: Order fields expression string.
+        :type fields: str
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.orderBy(fields))
 
@@ -449,7 +487,9 @@ class Table(object):
             >>> tab.order_by("name.desc").offset(10).fetch(5)
 
         :param offset: Number of records to skip.
-        :return: The result :class:`Table`.
+        :type offset: int
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.offset(offset))
 
@@ -474,7 +514,9 @@ class Table(object):
             >>> tab.order_by("name.desc").offset(10).fetch(5)
 
         :param fetch: The number of records to return. Fetch must be >= 0.
-        :return: The result :class:`Table`.
+        :type fetch: int
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.fetch(fetch))
 
@@ -506,12 +548,13 @@ class Table(object):
             ...     .group_by("w") \\
             ...     .select("a.sum as a, w.start as b, w.end as c, w.rowtime 
as d")
 
-        :param window: A :class:`pyflink.table.window.GroupWindow` created from
-                       :class:`pyflink.table.window.Tumble`, 
:class:`pyflink.table.window.Session`
-                       or :class:`pyflink.table.window.Slide`.
-        :return: A :class:`GroupWindowedTable`.
+        :param window: A :class:`~pyflink.table.window.GroupWindow` created 
from
+                       :class:`~pyflink.table.window.Tumble`, 
:class:`~pyflink.table.window.Session`
+                       or :class:`~pyflink.table.window.Slide`.
+        :type window: pyflink.table.window.GroupWindow
+        :return: A group windowed table.
+        :rtype: GroupWindowedTable
         """
-        # type: (GroupWindow) -> GroupWindowedTable
         return GroupWindowedTable(self._j_table.window(window._java_window))
 
     def over_window(self, *over_windows):
@@ -538,8 +581,10 @@ class Table(object):
 
             Over-windows for batch tables are currently not supported.
 
-        :param over_windows: :class:`OverWindow`s created from :class:`Over`.
-        :return: A :class:`OverWindowedTable`.
+        :param over_windows: over windows created from 
:class:`~pyflink.table.window.Over`.
+        :type over_windows: pyflink.table.window.OverWindow
+        :return: A over windowed table.
+        :rtype: pyflink.table.OverWindowedTable
         """
         gateway = get_gateway()
         window_array = to_jarray(gateway.jvm.OverWindow,
@@ -558,7 +603,9 @@ class Table(object):
             >>> tab.add_columns("a + 1 as a1, concat(b, 'sunny') as b1")
 
         :param fields: Column list string.
-        :return: The result :class:`Table`.
+        :type fields: str
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.addColumns(fields))
 
@@ -575,7 +622,9 @@ class Table(object):
             >>> tab.add_or_replace_columns("a + 1 as a1, concat(b, 'sunny') as 
b1")
 
         :param fields: Column list string.
-        :return: The result :class:`Table`.
+        :type fields: str
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.addOrReplaceColumns(fields))
 
@@ -590,7 +639,9 @@ class Table(object):
             >>> tab.rename_columns("a as a1, b as b1")
 
         :param fields: Column list string.
-        :return: The result :class:`Table`.
+        :type fields: str
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.renameColumns(fields))
 
@@ -604,14 +655,16 @@ class Table(object):
             >>> tab.drop_columns("a, b")
 
         :param fields: Column list string.
-        :return: The result :class:`Table`.
+        :type fields: str
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.dropColumns(fields))
 
     def insert_into(self, table_path):
         """
-        Writes the :class:`Table` to a :class:`TableSink` that was registered 
under
-        the specified name. For the path resolution algorithm see
+        Writes the :class:`~pyflink.table.Table` to a 
:class:`~pyflink.table.TableSink` that was
+        registered under the specified name. For the path resolution algorithm 
see
         :func:`~TableEnvironment.use_database`.
 
         Example:
@@ -619,16 +672,18 @@ class Table(object):
 
             >>> tab.insert_into("sink")
 
-        :param table_path: The path of the registered :class:`TableSink` to 
which
-               the :class:`Table` is written.
+        :param table_path: The path of the registered 
:class:`~pyflink.table.TableSink` to which
+               the :class:`~pyflink.table.Table` is written.
+        :type table_path: str
         """
         self._j_table.insertInto(table_path)
 
     def get_schema(self):
         """
-        Returns the :class:`TableSchema` of this table.
+        Returns the :class:`~pyflink.table.TableSchema` of this table.
 
         :return: The schema of this table.
+        :rtype: pyflink.table.TableSchema
         """
         return TableSchema(j_table_schema=self._j_table.getSchema())
 
@@ -662,14 +717,16 @@ class GroupedTable(object):
 
 
         :param fields: Expression string that contains group keys and 
aggregate function calls.
-        :return: The result :class:`Table`.
+        :type fields: str
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.select(fields))
 
 
 class GroupWindowedTable(object):
     """
-    A table that has been windowed for 
:class:`pyflink.table.window.GroupWindow`.
+    A table that has been windowed for :class:`~pyflink.table.GroupWindow`.
     """
 
     def __init__(self, java_group_windowed_table):
@@ -693,14 +750,16 @@ class GroupWindowedTable(object):
             >>> tab.window(group_window.alias("w")).group_by("w, 
key").select("key, value.avg")
 
         :param fields: Group keys.
-        :return: A :class:`WindowGroupedTable`.
+        :type fields: str
+        :return: A window grouped table.
+        :rtype: pyflink.table.WindowGroupedTable
         """
         return WindowGroupedTable(self._j_table.groupBy(fields))
 
 
 class WindowGroupedTable(object):
     """
-    A table that has been windowed and grouped for 
:class:`pyflink.table.window.GroupWindow`.
+    A table that has been windowed and grouped for 
:class:`~pyflink.table.window.GroupWindow`.
     """
 
     def __init__(self, java_window_grouped_table):
@@ -718,14 +777,16 @@ class WindowGroupedTable(object):
             >>> window_grouped_table.select("key, window.start, value.avg as 
valavg")
 
         :param fields: Expression string.
-        :return: The result :class:`Table`.
+        :type fields: str
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.select(fields))
 
 
 class OverWindowedTable(object):
     """
-    A table that has been windowed for 
:class:`pyflink.table.window.OverWindow`.
+    A table that has been windowed for 
:class:`~pyflink.table.window.OverWindow`.
 
     Unlike group windows, which are specified in the GROUP BY clause, over 
windows do not collapse
     rows. Instead over window aggregates compute an aggregate for each input 
row over a range of
@@ -747,6 +808,8 @@ class OverWindowedTable(object):
             >>> over_windowed_table.select("c, b.count over ow, e.sum over ow")
 
         :param fields: Expression string.
-        :return: The result :class:`Table`.
+        :type fields: str
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return Table(self._j_table.select(fields))
diff --git a/flink-python/pyflink/table/table_environment.py 
b/flink-python/pyflink/table/table_environment.py
index fc8ed83..68c7847 100644
--- a/flink-python/pyflink/table/table_environment.py
+++ b/flink-python/pyflink/table/table_environment.py
@@ -26,8 +26,7 @@ from pyflink.common.dependency_manager import 
DependencyManager
 from pyflink.serializers import BatchedSerializer, PickleSerializer
 from pyflink.table.catalog import Catalog
 from pyflink.table.table_config import TableConfig
-from pyflink.table.descriptors import (StreamTableDescriptor, 
ConnectorDescriptor,
-                                       BatchTableDescriptor)
+from pyflink.table.descriptors import StreamTableDescriptor, 
BatchTableDescriptor
 
 from pyflink.java_gateway import get_gateway
 from pyflink.table import Table
@@ -52,7 +51,8 @@ class TableEnvironment(object):
     A table environment is responsible for:
 
         - Connecting to external systems.
-        - Registering and retrieving :class:`Table` and other meta objects 
from a catalog.
+        - Registering and retrieving :class:`~pyflink.table.Table` and other 
meta objects from a
+          catalog.
         - Executing SQL statements.
         - Offering further configuration options.
 
@@ -94,27 +94,33 @@ class TableEnvironment(object):
             >>> table_env.from_table_source(csv_table_source)
 
         :param table_source: The table source used as table.
-        :return: The result :class:`Table`.
+        :type table_source: pyflink.table.TableSource
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         return 
Table(self._j_tenv.fromTableSource(table_source._j_table_source))
 
     def register_catalog(self, catalog_name, catalog):
         """
-        Registers a :class:`pyflink.table.catalog.Catalog` under a unique name.
-        All tables registered in the :class:`pyflink.table.catalog.Catalog` 
can be accessed.
+        Registers a :class:`~pyflink.table.catalog.Catalog` under a unique 
name.
+        All tables registered in the :class:`~pyflink.table.catalog.Catalog` 
can be accessed.
 
         :param catalog_name: The name under which the catalog will be 
registered.
-        :param catalog: The :class:`pyflink.table.catalog.Catalog` to register.
+        :type catalog_name: str
+        :param catalog: The catalog to register.
+        :type catalog: pyflink.table.catalog.Catalog
         """
         self._j_tenv.registerCatalog(catalog_name, catalog._j_catalog)
 
     def get_catalog(self, catalog_name):
         """
-        Gets a registered :class:`pyflink.table.catalog.Catalog` by name.
+        Gets a registered :class:`~pyflink.table.catalog.Catalog` by name.
 
-        :param catalog_name: The name to look up the 
:class:`pyflink.table.catalog.Catalog`.
-        :return: The requested :class:`pyflink.table.catalog.Catalog`, None if 
there is no
+        :param catalog_name: The name to look up the 
:class:`~pyflink.table.catalog.Catalog`.
+        :type catalog_name: str
+        :return: The requested catalog, None if there is no
                  registered catalog with given name.
+        :rtype: pyflink.table.catalog.Catalog
         """
         catalog = self._j_tenv.getCatalog(catalog_name)
         if catalog.isPresent():
@@ -124,8 +130,8 @@ class TableEnvironment(object):
 
     def register_table(self, name, table):
         """
-        Registers a :class:`Table` under a unique name in the 
TableEnvironment's catalog.
-        Registered tables can be referenced in SQL queries.
+        Registers a :class:`~pyflink.table.Table` under a unique name in the 
TableEnvironment's
+        catalog. Registered tables can be referenced in SQL queries.
 
         Example:
         ::
@@ -134,7 +140,9 @@ class TableEnvironment(object):
             >>> table_env.register_table("source", tab)
 
         :param name: The name under which the table will be registered.
+        :type name: str
         :param table: The table to register.
+        :type table: pyflink.table.Table
 
         .. note:: Deprecated in 1.10. Use :func:`create_temporary_view` 
instead.
         """
@@ -143,8 +151,9 @@ class TableEnvironment(object):
 
     def register_table_source(self, name, table_source):
         """
-        Registers an external :class:`TableSource` in this 
:class:`TableEnvironment`'s catalog.
-        Registered tables can be referenced in SQL queries.
+        Registers an external :class:`~pyflink.table.TableSource` in this
+        :class:`~pyflink.table.TableEnvironment`'s catalog. Registered tables 
can be referenced in
+        SQL queries.
 
         Example:
         ::
@@ -155,8 +164,10 @@ class TableEnvironment(object):
             ...                                                
[DataTypes.INT(),
             ...                                                 
DataTypes.STRING()]))
 
-        :param name: The name under which the :class:`TableSource` is 
registered.
-        :param table_source: The :class:`TableSource` to register.
+        :param name: The name under which the table source is registered.
+        :type name: str
+        :param table_source: The table source to register.
+        :type table_source: pyflink.table.TableSource
 
         .. note:: Deprecated in 1.10. Use :func:`connect` instead.
         """
@@ -165,9 +176,9 @@ class TableEnvironment(object):
 
     def register_table_sink(self, name, table_sink):
         """
-        Registers an external :class:`TableSink` with given field names and 
types in this
-        :class:`TableEnvironment`'s catalog.
-        Registered sink tables can be referenced in SQL DML statements.
+        Registers an external :class:`~pyflink.table.TableSink` with given 
field names and types in
+        this :class:`~pyflink.table.TableEnvironment`'s catalog. Registered 
sink tables can be
+        referenced in SQL DML statements.
 
         Example:
         ::
@@ -178,8 +189,10 @@ class TableEnvironment(object):
             ...                                             
DataTypes.STRING()],
             ...                                            "./2.csv"))
 
-        :param name: The name under which the :class:`TableSink` is registered.
-        :param table_sink: The :class:`TableSink` to register.
+        :param name: The name under which the table sink is registered.
+        :type name: str
+        :param table_sink: The table sink to register.
+        :type table_sink: pyflink.table.TableSink
 
         .. note:: Deprecated in 1.10. Use :func:`connect` instead.
         """
@@ -188,9 +201,9 @@ class TableEnvironment(object):
 
     def scan(self, *table_path):
         """
-        Scans a registered table and returns the resulting :class:`Table`.
+        Scans a registered table and returns the resulting 
:class:`~pyflink.table.Table`.
         A table to scan must be registered in the TableEnvironment. It can be 
either directly
-        registered or be an external member of a 
:class:`pyflink.table.catalog.Catalog`.
+        registered or be an external member of a 
:class:`~pyflink.table.catalog.Catalog`.
 
         See the documentation of 
:func:`~pyflink.table.TableEnvironment.use_database` or
         :func:`~pyflink.table.TableEnvironment.use_catalog` for the rules on 
the path resolution.
@@ -208,8 +221,10 @@ class TableEnvironment(object):
             >>> tab = table_env.scan("catalogName", "dbName", "tableName")
 
         :param table_path: The path of the table to scan.
+        :type table_path: str
         :throws: Exception if no table is found using the given table path.
-        :return: The resulting :class:`Table`
+        :return: The resulting table.
+        :rtype: pyflink.table.Table
 
         .. note:: Deprecated in 1.10. Use :func:`from_path` instead.
         """
@@ -221,9 +236,9 @@ class TableEnvironment(object):
 
     def from_path(self, path):
         """
-        Reads a registered table and returns the resulting :class:`Table`.
+        Reads a registered table and returns the resulting 
:class:`~pyflink.table.Table`.
 
-        A table to scan must be registered in the :class:`TableEnvironment`.
+        A table to scan must be registered in the 
:class:`~pyflink.table.TableEnvironment`.
 
         See the documentation of :func:`use_database` or :func:`use_catalog` 
for the rules on the
         path resolution.
@@ -247,16 +262,18 @@ class TableEnvironment(object):
             >>> tab = table_env.from_path("catalogName.`db.Name`.`Table`")
 
         :param path: The path of a table API object to scan.
+        :type path: str
         :return: Either a table or virtual table (=view).
+        :rtype: pyflink.table.Table
 
-        .. seealso:: :func:`user_catalog`
-        .. seealso:: :func:`user_database`
+        .. seealso:: :func:`use_catalog`
+        .. seealso:: :func:`use_database`
         """
         return Table(get_method(self._j_tenv, "from")(path))
 
     def insert_into(self, target_path, table):
         """
-        Instructs to write the content of a :class:`Table` API object into a 
table.
+        Instructs to write the content of a :class:`~pyflink.table.Table` API 
object into a table.
 
         See the documentation of :func:`use_database` or :func:`use_catalog` 
for the rules on the
         path resolution.
@@ -267,9 +284,11 @@ class TableEnvironment(object):
             >>> tab = table_env.scan("tableName")
             >>> table_env.insert_into("sink", tab)
 
-        :param target_path: The path of the registered :class:`TableSink` to 
which the
-                            :class:`Table` is written.
+        :param target_path: The path of the registered 
:class:`~pyflink.table.TableSink` to which
+                            the :class:`~pyflink.table.Table` is written.
+        :type target_path: str
         :param table: table The Table to write to the sink.
+        :type table: pyflink.table.Table
         """
         self._j_tenv.insertInto(target_path, table._j_table)
 
@@ -278,6 +297,7 @@ class TableEnvironment(object):
         Gets the names of all catalogs registered in this environment.
 
         :return: List of catalog names.
+        :rtype: list[str]
         """
         j_catalog_name_array = self._j_tenv.listCatalogs()
         return [item for item in j_catalog_name_array]
@@ -287,6 +307,7 @@ class TableEnvironment(object):
         Gets the names of all modules registered in this environment.
 
         :return: List of module names.
+        :rtype: list[str]
         """
         j_module_name_array = self._j_tenv.listModules()
         return [item for item in j_module_name_array]
@@ -296,6 +317,7 @@ class TableEnvironment(object):
         Gets the names of all databases in the current catalog.
 
         :return: List of database names in the current catalog.
+        :rtype: list[str]
         """
         j_database_name_array = self._j_tenv.listDatabases()
         return [item for item in j_database_name_array]
@@ -305,6 +327,7 @@ class TableEnvironment(object):
         Gets the names of all tables in the current database of the current 
catalog.
 
         :return: List of table names in the current database of the current 
catalog.
+        :rtype: list[str]
         """
         j_table_name_array = self._j_tenv.listTables()
         return [item for item in j_table_name_array]
@@ -336,6 +359,7 @@ class TableEnvironment(object):
 
         :return: A list of the names of all registered temporary tables and 
views in the current
                  database of the current catalog.
+        :rtype: list[str]
 
         .. seealso:: :func:`list_tables`
         """
@@ -349,6 +373,7 @@ class TableEnvironment(object):
 
         :return: A list of the names of all registered temporary views in the 
current database
                  of the current catalog.
+        :rtype: list[str]
 
         .. seealso:: :func:`list_tables`
         """
@@ -362,7 +387,10 @@ class TableEnvironment(object):
         If a permanent table with a given path exists, it will be used
         from now on for any queries that reference this path.
 
-        :return: true if a table existed in the given path and was removed
+        :param table_path: The path of the registered temporary table.
+        :type table_path: str
+        :return: True if a table existed in the given path and was removed.
+        :rtype: bool
         """
         return self._j_tenv.dropTemporaryTable(table_path)
 
@@ -373,20 +401,24 @@ class TableEnvironment(object):
         If a permanent table or view with a given path exists, it will be used
         from now on for any queries that reference this path.
 
-        :return: true if a view existed in the given path and was removed
+        :return: True if a view existed in the given path and was removed.
+        :rtype: bool
         """
         return self._j_tenv.dropTemporaryView(view_path)
 
     def explain(self, table=None, extended=False):
         """
         Returns the AST of the specified Table API and SQL queries and the 
execution plan to compute
-        the result of the given :class:`Table` or multi-sinks plan.
+        the result of the given :class:`~pyflink.table.Table` or multi-sinks 
plan.
 
         :param table: The table to be explained. If table is None, explain for 
multi-sinks plan,
                       else for given table.
+        :type table: pyflink.table.Table
         :param extended: If the plan should contain additional properties.
                          e.g. estimated cost, traits
+        :type extended: bool
         :return: The table for which the AST and execution plan will be 
returned.
+        :rtype: str
         """
         if table is None:
             return self._j_tenv.explain(extended)
@@ -395,14 +427,15 @@ class TableEnvironment(object):
 
     def sql_query(self, query):
         """
-        Evaluates a SQL query on registered tables and retrieves the result as 
a :class:`Table`.
+        Evaluates a SQL query on registered tables and retrieves the result as 
a
+        :class:`~pyflink.table.Table`.
 
         All tables referenced by the query must be registered in the 
TableEnvironment.
 
-        A :class:`Table` is automatically registered when its 
:func:`~Table.__str__` method is
-        called, for example when it is embedded into a String.
+        A :class:`~pyflink.table.Table` is automatically registered when its
+        :func:`~Table.__str__` method is called, for example when it is 
embedded into a String.
 
-        Hence, SQL queries can directly reference a :class:`Table` as follows:
+        Hence, SQL queries can directly reference a 
:class:`~pyflink.table.Table` as follows:
         ::
 
             >>> table = ...
@@ -410,7 +443,9 @@ class TableEnvironment(object):
             >>> table_env.sql_query("SELECT * FROM %s" % table)
 
         :param query: The sql query string.
-        :return: The result :class:`Table`.
+        :type query: str
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
         j_table = self._j_tenv.sqlQuery(query)
         return Table(j_table)
@@ -424,9 +459,9 @@ class TableEnvironment(object):
             Currently only SQL INSERT statements and CREATE TABLE statements 
are supported.
 
         All tables referenced by the query must be registered in the 
TableEnvironment.
-        A :class:`Table` is automatically registered when its 
:func:`~Table.__str__` method is
-        called, for example when it is embedded into a String.
-        Hence, SQL queries can directly reference a :class:`Table` as follows:
+        A :class:`~pyflink.table.Table` is automatically registered when its
+        :func:`~Table.__str__` method is called, for example when it is 
embedded into a String.
+        Hence, SQL queries can directly reference a 
:class:`~pyflink.table.Table` as follows:
         ::
 
             # register the table sink into which the result is inserted.
@@ -485,9 +520,8 @@ class TableEnvironment(object):
             >>> table_env.execute("MyJob")
 
         :param stmt: The SQL statement to evaluate.
-        :param query_config: The :class:`QueryConfig` to use.
+        :type stmt: str
         """
-        # type: (str) -> None
         self._j_tenv.sqlUpdate(stmt)
 
     def get_current_catalog(self):
@@ -495,6 +529,7 @@ class TableEnvironment(object):
         Gets the current default catalog name of the current session.
 
         :return: The current default catalog name that is used for the path 
resolution.
+        :rtype: str
 
         .. seealso:: :func:`~pyflink.table.TableEnvironment.use_catalog`
         """
@@ -541,8 +576,9 @@ class TableEnvironment(object):
         | cat1.db1.tab1  | cat1.db1.tab1                           |
         +----------------+-----------------------------------------+
 
-        :param: catalog_name: The name of the catalog to set as the current 
default catalog.
-        :throws: :class:`pyflink.util.exceptions.CatalogException` thrown if a 
catalog with given
+        :param catalog_name: The name of the catalog to set as the current 
default catalog.
+        :type catalog_name: str
+        :throws: :class:`~pyflink.util.exceptions.CatalogException` thrown if 
a catalog with given
                  name could not be set as the default one.
 
         .. seealso:: :func:`~pyflink.table.TableEnvironment.use_database`
@@ -554,6 +590,7 @@ class TableEnvironment(object):
         Gets the current default database name of the running session.
 
         :return: The name of the current database of the current catalog.
+        :rtype: str
 
         .. seealso:: :func:`~pyflink.table.TableEnvironment.use_database`
         """
@@ -599,12 +636,13 @@ class TableEnvironment(object):
         | cat1.db1.tab1  | cat1.db1.tab1                           |
         +----------------+-----------------------------------------+
 
-        :throws: :class:`pyflink.util.exceptions.CatalogException` thrown if 
the given catalog and
+        :throws: :class:`~pyflink.util.exceptions.CatalogException` thrown if 
the given catalog and
                  database could not be set as the default ones.
 
         .. seealso:: :func:`~pyflink.table.TableEnvironment.use_catalog`
 
-        :param: database_name: The name of the database to set as the current 
database.
+        :param database_name: The name of the database to set as the current 
database.
+        :type database_name: str
         """
         self._j_tenv.useDatabase(database_name)
 
@@ -613,7 +651,8 @@ class TableEnvironment(object):
         """
         Returns the table config to define the runtime behavior of the Table 
API.
 
-        :return: Current :class:`TableConfig`.
+        :return: Current table config.
+        :rtype: pyflink.table.TableConfig
         """
         pass
 
@@ -645,8 +684,10 @@ class TableEnvironment(object):
             ...     .register_table_source("MyTable")
 
         :param connector_descriptor: Connector descriptor describing the 
external system.
-        :return: A :class:`pyflink.table.descriptors.ConnectTableDescriptor` 
used to build the
+        :type connector_descriptor: 
pyflink.table.descriptors.ConnectorDescriptor
+        :return: A :class:`~pyflink.table.descriptors.ConnectTableDescriptor` 
used to build the
                  table source/sink.
+        :rtype: pyflink.table.descriptors.ConnectTableDescriptor
         """
         pass
 
@@ -699,22 +740,26 @@ class TableEnvironment(object):
         :param name: The name under which the function is registered.
         :type name: str
         :param function: The python user-defined function to register.
-        :type function: UserDefinedFunctionWrapper
+        :type function: pyflink.table.udf.UserDefinedFunctionWrapper
         """
         self._j_tenv.registerFunction(name, 
function._judf(self._is_blink_planner,
                                                            
self.get_config()._j_table_config))
 
     def create_temporary_view(self, view_path, table):
         """
-        Registers a :class:`Table` API object as a temporary view similar to 
SQL temporary views.
+        Registers a :class:`~pyflink.table.Table` API object as a temporary 
view similar to SQL
+        temporary views.
 
         Temporary objects can shadow permanent ones. If a permanent object in 
a given path exists,
         it will be inaccessible in the current session. To make the permanent 
object available
         again you can drop the corresponding temporary object.
 
         :param view_path: The path under which the view will be registered. 
See also the
-                          :class:`TableEnvironment` class description for the 
format of the path.
+                          :class:`~pyflink.table.TableEnvironment` class 
description for the format
+                          of the path.
+        :type view_path: str
         :param table: The view to register.
+        :type table: pyflink.table.Table
         """
         self._j_tenv.createTemporaryView(view_path, table._j_table)
 
@@ -826,7 +871,7 @@ class TableEnvironment(object):
 
         .. note::
 
-            It is highly advised to set all parameters in the 
:class:`TableConfig`
+            It is highly advised to set all parameters in the 
:class:`~pyflink.table.TableConfig`
             on the very beginning of the program. It is undefined what 
configurations values will
             be used for the execution if queries are mixed with config 
changes. It depends on
             the characteristic of the particular parameter. For some of them 
the value from the
@@ -857,7 +902,7 @@ class TableEnvironment(object):
 
         The built-in acceptable composite element types contains:
 
-        **list**, **tuple**, **dict**, **array**, :class:`pyflink.table.Row`
+        **list**, **tuple**, **dict**, **array**, :class:`~pyflink.table.Row`
 
         If the element type is a composite type, it will be unboxed.
         e.g. table_env.from_elements([(1, 'Hi'), (2, 'Hello')]) will return a 
table like:
@@ -888,9 +933,13 @@ class TableEnvironment(object):
             ...                         False)
 
         :param elements: The elements to create a table from.
+        :type elements: list
         :param schema: The schema of the table.
+        :type schema: pyflink.table.types.DataType
         :param verify_schema: Whether to verify the elements against the 
schema.
-        :return: The result :class:`Table`.
+        :type verify_schema: bool
+        :return: The result table.
+        :rtype: pyflink.table.Table
         """
 
         # verifies the elements against the specified schema
@@ -942,7 +991,7 @@ class TableEnvironment(object):
         Creates a table from a collection of elements.
 
         :param elements: The elements to create a table from.
-        :return: The result :class:`Table`.
+        :return: The result :class:`~pyflink.table.Table`.
         """
 
         # serializes to a file, and we read the file in java
@@ -992,7 +1041,8 @@ class StreamTableEnvironment(TableEnvironment):
         """
         Returns the table config to define the runtime behavior of the Table 
API.
 
-        :return: Current :class:`TableConfig`.
+        :return: Current table config.
+        :rtype: pyflink.table.TableConfig
         """
         table_config = TableConfig()
         table_config._j_table_config = self._j_tenv.getConfig()
@@ -1023,16 +1073,18 @@ class StreamTableEnvironment(TableEnvironment):
             ...     .register_table_source("MyTable")
 
         :param connector_descriptor: Connector descriptor describing the 
external system.
-        :return: A :class:`StreamTableDescriptor` used to build the table 
source/sink.
+        :type connector_descriptor: 
pyflink.table.descriptors.ConnectorDescriptor
+        :return: A :class:`~pyflink.table.descriptors.StreamTableDescriptor` 
used to build the table
+                 source/sink.
+        :rtype: pyflink.table.descriptors.StreamTableDescriptor
         """
-        # type: (ConnectorDescriptor) -> StreamTableDescriptor
         return StreamTableDescriptor(
             self._j_tenv.connect(connector_descriptor._j_connector_descriptor))
 
     @staticmethod
     def create(stream_execution_environment, table_config=None, 
environment_settings=None):
         """
-        Creates a :class:`TableEnvironment` for a
+        Creates a :class:`~pyflink.table.TableEnvironment` for a
         :class:`~pyflink.datastream.StreamExecutionEnvironment`.
 
         Example:
@@ -1057,14 +1109,14 @@ class StreamTableEnvironment(TableEnvironment):
                                              of the TableEnvironment.
         :type stream_execution_environment: 
pyflink.datastream.StreamExecutionEnvironment
         :param table_config: The configuration of the TableEnvironment, 
optional.
-        :type table_config: TableConfig
+        :type table_config: pyflink.table.TableConfig
         :param environment_settings: The environment settings used to 
instantiate the
                                      TableEnvironment. It provides the 
interfaces about planner
                                      selection(flink or blink), optional.
         :type environment_settings: pyflink.table.EnvironmentSettings
-        :return: The :class:`StreamTableEnvironment` created from given 
StreamExecutionEnvironment
-                 and configuration.
-        :rtype: StreamTableEnvironment
+        :return: The StreamTableEnvironment created from given 
StreamExecutionEnvironment and
+                 configuration.
+        :rtype: pyflink.table.StreamTableEnvironment
         """
         if table_config is not None and environment_settings is not None:
             raise ValueError("The param 'table_config' and "
@@ -1108,7 +1160,8 @@ class BatchTableEnvironment(TableEnvironment):
         """
         Returns the table config to define the runtime behavior of the Table 
API.
 
-        :return: Current :class:`TableConfig`.
+        :return: Current table config.
+        :rtype: pyflink.table.TableConfig
         """
         table_config = TableConfig()
         table_config._j_table_config = self._j_tenv.getConfig()
@@ -1139,10 +1192,12 @@ class BatchTableEnvironment(TableEnvironment):
             ...     .register_table_source("MyTable")
 
         :param connector_descriptor: Connector descriptor describing the 
external system.
-        :type connector_descriptor: ConnectorDescriptor
-        :return: A :class:`BatchTableDescriptor` or a 
:class:`StreamTableDescriptor`
-                 (for blink planner) used to build the table source/sink.
-        :rtype: BatchTableDescriptor or StreamTableDescriptor
+        :type connector_descriptor: 
pyflink.table.descriptors.ConnectorDescriptor
+        :return: A :class:`~pyflink.table.descriptors.BatchTableDescriptor` or 
a
+                 :class:`~pyflink.table.descriptors.StreamTableDescriptor` 
(for blink planner) used
+                 to build the table source/sink.
+        :rtype: pyflink.table.descriptors.BatchTableDescriptor or
+                pyflink.table.descriptors.StreamTableDescriptor
         """
         gateway = get_gateway()
         blink_t_env_class = get_java_class(
@@ -1157,7 +1212,7 @@ class BatchTableEnvironment(TableEnvironment):
     @staticmethod
     def create(execution_environment=None, table_config=None, 
environment_settings=None):
         """
-        Creates a :class:`BatchTableEnvironment`.
+        Creates a :class:`~pyflink.table.BatchTableEnvironment`.
 
         Example:
         ::
@@ -1174,18 +1229,18 @@ class BatchTableEnvironment(TableEnvironment):
             ...     .use_blink_planner().build()
             >>> table_env = 
BatchTableEnvironment.create(environment_settings=environment_settings)
 
-        :param execution_environment: The batch 
:class:`pyflink.dataset.ExecutionEnvironment` of
+        :param execution_environment: The batch 
:class:`~pyflink.dataset.ExecutionEnvironment` of
                                       the TableEnvironment.
         :type execution_environment: pyflink.dataset.ExecutionEnvironment
         :param table_config: The configuration of the TableEnvironment, 
optional.
-        :type table_config: TableConfig
+        :type table_config: pyflink.table.TableConfig
         :param environment_settings: The environment settings used to 
instantiate the
                                      TableEnvironment. It provides the 
interfaces about planner
                                      selection(flink or blink), optional.
         :type environment_settings: pyflink.table.EnvironmentSettings
         :return: The BatchTableEnvironment created from given 
ExecutionEnvironment and
                  configuration.
-        :rtype: BatchTableEnvironment
+        :rtype: pyflink.table.BatchTableEnvironment
         """
         if execution_environment is None and \
                 table_config is None and \

Reply via email to