]> granicus.if.org Git - postgresql/blobdiff - src/include/nodes/relation.h
Add a Gather executor node.
[postgresql] / src / include / nodes / relation.h
index 4f1bc4067d21f9ea5aedd47a676b562033d3f35e..6cf2e24ce7d30e06cb759d32f64962723ef28dd5 100644 (file)
@@ -4,10 +4,10 @@
  *       Definitions for planner's internal data structures.
  *
  *
- * Portions Copyright (c) 1996-2009, PostgreSQL Global Development Group
+ * Portions Copyright (c) 1996-2015, PostgreSQL Global Development Group
  * Portions Copyright (c) 1994, Regents of the University of California
  *
- * $PostgreSQL: pgsql/src/include/nodes/relation.h,v 1.170 2009/03/05 23:06:45 tgl Exp $
+ * src/include/nodes/relation.h
  *
  *-------------------------------------------------------------------------
  */
@@ -15,7 +15,7 @@
 #define RELATION_H
 
 #include "access/sdir.h"
-#include "nodes/bitmapset.h"
+#include "lib/stringinfo.h"
 #include "nodes/params.h"
 #include "nodes/parsenodes.h"
 #include "storage/block.h"
@@ -46,6 +46,21 @@ typedef struct QualCost
        Cost            per_tuple;              /* per-evaluation cost */
 } QualCost;
 
+/*
+ * Costing aggregate function execution requires these statistics about
+ * the aggregates to be executed by a given Agg node.  Note that the costs
+ * include the execution costs of the aggregates' argument expressions as
+ * well as the aggregate functions themselves.
+ */
+typedef struct AggClauseCosts
+{
+       int                     numAggs;                /* total number of aggregate functions */
+       int                     numOrderedAggs; /* number w/ DISTINCT/ORDER BY/WITHIN GROUP */
+       QualCost        transCost;              /* total per-input-row execution costs */
+       Cost            finalCost;              /* total per-aggregated-row costs */
+       Size            transitionSpace;        /* space for pass-by-ref transition data */
+} AggClauseCosts;
+
 
 /*----------
  * PlannerGlobal
@@ -62,23 +77,37 @@ typedef struct PlannerGlobal
 
        ParamListInfo boundParams;      /* Param values provided to planner() */
 
-       List       *paramlist;          /* to keep track of cross-level Params */
-
        List       *subplans;           /* Plans for SubPlan nodes */
 
-       List       *subrtables;         /* Rangetables for SubPlan nodes */
+       List       *subroots;           /* PlannerInfos for SubPlan nodes */
 
        Bitmapset  *rewindPlanIDs;      /* indices of subplans that require REWIND */
 
        List       *finalrtable;        /* "flat" rangetable for executor */
 
+       List       *finalrowmarks;      /* "flat" list of PlanRowMarks */
+
+       List       *resultRelations;    /* "flat" list of integer RT indexes */
+
        List       *relationOids;       /* OIDs of relations the plan depends on */
 
        List       *invalItems;         /* other dependencies, as PlanInvalItems */
 
+       int                     nParamExec;             /* number of PARAM_EXEC Params used */
+
        Index           lastPHId;               /* highest PlaceHolderVar ID assigned */
 
+       Index           lastRowMarkId;  /* highest PlanRowMark ID assigned */
+
+       int                     lastPlanNodeId; /* highest plan node ID assigned */
+
        bool            transientPlan;  /* redo plan when TransactionXmin changes? */
+
+       bool            hasRowSecurity; /* row security applied? */
+
+       bool            parallelModeOK; /* parallel mode potentially OK? */
+
+       bool            parallelModeNeeded;     /* parallel mode actually required? */
 } PlannerGlobal;
 
 /* macro for fetching the Plan associated with a SubPlan node */
@@ -92,7 +121,7 @@ typedef struct PlannerGlobal
  *
  * This struct is conventionally called "root" in all the planner routines.
  * It holds links to all of the planner's working state, in addition to the
- * original Query.     Note that at present the planner extensively modifies
+ * original Query.  Note that at present the planner extensively modifies
  * the passed-in Query data structure; someday that should stop.
  *----------
  */
@@ -106,11 +135,20 @@ typedef struct PlannerInfo
 
        Index           query_level;    /* 1 at the outermost Query */
 
-       struct PlannerInfo *parent_root; /* NULL at outermost Query */
+       struct PlannerInfo *parent_root;        /* NULL at outermost Query */
+
+       /*
+        * plan_params contains the expressions that this query level needs to
+        * make available to a lower query level that is currently being planned.
+        * outer_params contains the paramIds of PARAM_EXEC Params that outer
+        * query levels will make available to this query level.
+        */
+       List       *plan_params;        /* list of PlannerParamItems, see below */
+       Bitmapset  *outer_params;
 
        /*
         * simple_rel_array holds pointers to "base rels" and "other rels" (see
-        * comments for RelOptInfo for more info).      It is indexed by rangetable
+        * comments for RelOptInfo for more info).  It is indexed by rangetable
         * index (so entry 0 is always wasted).  Entries can be NULL when an RTE
         * does not correspond to a base relation, such as a join RTE or an
         * unreferenced view RTE; or if the RelOptInfo hasn't been made yet.
@@ -126,26 +164,51 @@ typedef struct PlannerInfo
         */
        RangeTblEntry **simple_rte_array;       /* rangetable as an array */
 
+       /*
+        * all_baserels is a Relids set of all base relids (but not "other"
+        * relids) in the query; that is, the Relids identifier of the final join
+        * we need to form.  This is computed in make_one_rel, just before we
+        * start making Paths.
+        */
+       Relids          all_baserels;
+
+       /*
+        * nullable_baserels is a Relids set of base relids that are nullable by
+        * some outer join in the jointree; these are rels that are potentially
+        * nullable below the WHERE clause, SELECT targetlist, etc.  This is
+        * computed in deconstruct_jointree.
+        */
+       Relids          nullable_baserels;
+
        /*
         * join_rel_list is a list of all join-relation RelOptInfos we have
         * considered in this planning run.  For small problems we just scan the
         * list to do lookups, but when there are many join relations we build a
         * hash table for faster lookups.  The hash table is present and valid
-        * when join_rel_hash is not NULL.      Note that we still maintain the list
+        * when join_rel_hash is not NULL.  Note that we still maintain the list
         * even when using the hash table for lookups; this simplifies life for
         * GEQO.
         */
        List       *join_rel_list;      /* list of join-relation RelOptInfos */
        struct HTAB *join_rel_hash; /* optional hashtable for join relations */
 
-       List       *resultRelations;    /* integer list of RT indexes, or NIL */
-
-       List       *returningLists; /* list of lists of TargetEntry, or NIL */
+       /*
+        * When doing a dynamic-programming-style join search, join_rel_level[k]
+        * is a list of all join-relation RelOptInfos of level k, and
+        * join_cur_level is the current level.  New join-relation RelOptInfos are
+        * automatically added to the join_rel_level[join_cur_level] list.
+        * join_rel_level is NULL if not in use.
+        */
+       List      **join_rel_level; /* lists of join-relation RelOptInfos */
+       int                     join_cur_level; /* index of list being extended */
 
        List       *init_plans;         /* init SubPlans for query */
 
        List       *cte_plan_ids;       /* per-CTE-item list of subplan IDs */
 
+       List       *multiexpr_params;           /* List of Lists of Params for
+                                                                                * MULTIEXPR subquery outputs */
+
        List       *eq_classes;         /* list of active EquivalenceClasses */
 
        List       *canon_pathkeys; /* list of "canonical" PathKeys */
@@ -161,19 +224,25 @@ typedef struct PlannerInfo
        List       *full_join_clauses;          /* list of RestrictInfos for
                                                                                 * mergejoinable full join clauses */
 
-       List       *join_info_list;             /* list of SpecialJoinInfos */
+       List       *join_info_list; /* list of SpecialJoinInfos */
+
+       List       *lateral_info_list;          /* list of LateralJoinInfos */
 
        List       *append_rel_list;    /* list of AppendRelInfos */
 
-       List       *placeholder_list;   /* list of PlaceHolderInfos */
+       List       *rowMarks;           /* list of PlanRowMarks */
+
+       List       *placeholder_list;           /* list of PlaceHolderInfos */
 
        List       *query_pathkeys; /* desired pathkeys for query_planner(), and
-                                                                * actual pathkeys afterwards */
+                                                                * actual pathkeys after planning */
 
-       List       *group_pathkeys;             /* groupClause pathkeys, if any */
+       List       *group_pathkeys; /* groupClause pathkeys, if any */
        List       *window_pathkeys;    /* pathkeys of bottom window, if any */
-       List       *distinct_pathkeys;  /* distinctClause pathkeys, if any */
-       List       *sort_pathkeys;              /* sortClause pathkeys, if any */
+       List       *distinct_pathkeys;          /* distinctClause pathkeys, if any */
+       List       *sort_pathkeys;      /* sortClause pathkeys, if any */
+
+       List       *minmax_aggs;        /* List of MinMaxAggInfos */
 
        List       *initial_rels;       /* RelOptInfos we are now trying to join */
 
@@ -182,16 +251,31 @@ typedef struct PlannerInfo
        double          total_table_pages;              /* # of pages in all tables of query */
 
        double          tuple_fraction; /* tuple_fraction passed to query_planner */
+       double          limit_tuples;   /* limit_tuples passed to query_planner */
 
+       bool            hasInheritedTarget;             /* true if parse->resultRelation is an
+                                                                                * inheritance child rel */
        bool            hasJoinRTEs;    /* true if any RTEs are RTE_JOIN kind */
+       bool            hasLateralRTEs; /* true if any RTEs are marked LATERAL */
+       bool            hasDeletedRTEs; /* true if any RTE was deleted from jointree */
        bool            hasHavingQual;  /* true if havingQual was non-null */
        bool            hasPseudoConstantQuals; /* true if any RestrictInfo has
                                                                                 * pseudoconstant = true */
        bool            hasRecursion;   /* true if planning a recursive WITH item */
 
        /* These fields are used only when hasRecursion is true: */
-       int                     wt_param_id;                    /* PARAM_EXEC ID for the work table */
+       int                     wt_param_id;    /* PARAM_EXEC ID for the work table */
        struct Plan *non_recursive_plan;        /* plan for non-recursive term */
+
+       /* These fields are workspace for createplan.c */
+       Relids          curOuterRels;   /* outer rels above current node */
+       List       *curOuterParams; /* not-yet-assigned NestLoopParams */
+
+       /* optional private data for join_search_hook, e.g., GEQO */
+       void       *join_search_private;
+
+       /* for GroupingFunc fixup in setrefs */
+       AttrNumber *grouping_map;
 } PlannerInfo;
 
 
@@ -223,24 +307,26 @@ typedef struct PlannerInfo
  *
  * We also have "other rels", which are like base rels in that they refer to
  * single RT indexes; but they are not part of the join tree, and are given
- * a different RelOptKind to identify them.
+ * a different RelOptKind to identify them.  Lastly, there is a RelOptKind
+ * for "dead" relations, which are base rels that we have proven we don't
+ * need to join after all.
  *
  * Currently the only kind of otherrels are those made for member relations
  * of an "append relation", that is an inheritance set or UNION ALL subquery.
  * An append relation has a parent RTE that is a base rel, which represents
- * the entire append relation. The member RTEs are otherrels.  The parent
+ * the entire append relation.  The member RTEs are otherrels.  The parent
  * is present in the query join tree but the members are not.  The member
  * RTEs and otherrels are used to plan the scans of the individual tables or
- * subqueries of the append set; then the parent baserel is given an Append
- * plan comprising the best plans for the individual member rels.  (See
- * comments for AppendRelInfo for more information.)
+ * subqueries of the append set; then the parent baserel is given Append
+ * and/or MergeAppend paths comprising the best paths for the individual
+ * member rels.  (See comments for AppendRelInfo for more information.)
  *
  * At one time we also made otherrels to represent join RTEs, for use in
  * handling join alias Vars.  Currently this is not needed because all join
  * alias Vars are expanded to non-aliased form during preprocess_expression.
  *
  * Parts of this data structure are specific to various scan and join
- * mechanisms. It didn't seem worth creating new node types for them.
+ * mechanisms.  It didn't seem worth creating new node types for them.
  *
  *             relids - Set of base-relation identifiers; it is a base relation
  *                             if there is just one, a join relation if more than one
@@ -248,20 +334,29 @@ typedef struct PlannerInfo
  *                        clauses have been applied (ie, output rows of a plan for it)
  *             width - avg. number of bytes per tuple in the relation after the
  *                             appropriate projections have been done (ie, output width)
+ *             consider_startup - true if there is any value in keeping plain paths for
+ *                                                this rel on the basis of having cheap startup cost
+ *             consider_param_startup - the same for parameterized paths
  *             reltargetlist - List of Var and PlaceHolderVar nodes for the values
  *                                             we need to output from this relation.
  *                                             List is in no particular order, but all rels of an
  *                                             appendrel set must use corresponding orders.
- *                                             NOTE: in a child relation, may contain RowExpr or
- *                                             ConvertRowtypeExpr representing a whole-row Var.
+ *                                             NOTE: in an appendrel child relation, may contain
+ *                                             arbitrary expressions pulled up from a subquery!
  *             pathlist - List of Path nodes, one for each potentially useful
  *                                method of generating the relation
+ *             ppilist - ParamPathInfo nodes for parameterized Paths, if any
  *             cheapest_startup_path - the pathlist member with lowest startup cost
- *                                                             (regardless of its ordering)
+ *                     (regardless of ordering) among the unparameterized paths;
+ *                     or NULL if there is no unparameterized path
  *             cheapest_total_path - the pathlist member with lowest total cost
- *                                                       (regardless of its ordering)
+ *                     (regardless of ordering) among the unparameterized paths;
+ *                     or if there is no unparameterized path, the path with lowest
+ *                     total cost among the paths with minimum parameterization
  *             cheapest_unique_path - for caching cheapest path to produce unique
- *                                                        (no duplicates) output from relation
+ *                     (no duplicates) output from relation; NULL if not yet requested
+ *             cheapest_parameterized_paths - best paths for their parameterizations;
+ *                     always includes cheapest_total_path, even if that's unparameterized
  *
  * If the relation is a base relation it will have these fields set:
  *
@@ -274,20 +369,34 @@ typedef struct PlannerInfo
  *                             the attribute is needed as part of final targetlist
  *             attr_widths - cache space for per-attribute width estimates;
  *                                       zero means not computed yet
+ *             lateral_vars - lateral cross-references of rel, if any (list of
+ *                                        Vars and PlaceHolderVars)
+ *             lateral_relids - required outer rels for LATERAL, as a Relids set
+ *                                              (for child rels this can be more than lateral_vars)
+ *             lateral_referencers - relids of rels that reference this one laterally
  *             indexlist - list of IndexOptInfo nodes for relation's indexes
  *                                     (always NIL if it's not a table)
  *             pages - number of disk pages in relation (zero if not a table)
  *             tuples - number of tuples in relation (not considering restrictions)
+ *             allvisfrac - fraction of disk pages that are marked all-visible
  *             subplan - plan for subquery (NULL if it's not a subquery)
- *             subrtable - rangetable for subquery (NIL if it's not a subquery)
+ *             subroot - PlannerInfo for subquery (NULL if it's not a subquery)
+ *             subplan_params - list of PlannerParamItems to be passed to subquery
  *
- *             Note: for a subquery, tuples and subplan are not set immediately
+ *             Note: for a subquery, tuples, subplan, subroot are not set immediately
  *             upon creation of the RelOptInfo object; they are filled in when
- *             set_base_rel_pathlist processes the object.
+ *             set_subquery_pathlist processes the object.
  *
  *             For otherrels that are appendrel members, these fields are filled
  *             in just as for a baserel.
  *
+ * If the relation is either a foreign table or a join of foreign tables that
+ * all belong to the same foreign server, these fields will be set:
+ *
+ *             serverid - OID of foreign server, if foreign table (else InvalidOid)
+ *             fdwroutine - function hooks for FDW, if foreign table (else NULL)
+ *             fdw_private - private state for FDW, if foreign table (else NULL)
+ *
  * The presence of the remaining fields depends on the restrictions
  * and joins that the relation participates in:
  *
@@ -301,11 +410,6 @@ typedef struct PlannerInfo
  *                                     note this excludes clauses that might be derivable from
  *                                     EquivalenceClasses)
  *             has_eclass_joins - flag that EquivalenceClass joins are possible
- *             index_outer_relids - only used for base rels; set of outer relids
- *                                     that participate in indexable joinclauses for this rel
- *             index_inner_paths - only used for base rels; list of InnerIndexscanInfo
- *                                     nodes showing best indexpaths for various subsets of
- *                                     index_outer_relids.
  *
  * Note: Keeping a restrictinfo list in the RelOptInfo is useful only for
  * base rels, because for a join rel the set of clauses that are treated as
@@ -328,7 +432,8 @@ typedef enum RelOptKind
 {
        RELOPT_BASEREL,
        RELOPT_JOINREL,
-       RELOPT_OTHER_MEMBER_REL
+       RELOPT_OTHER_MEMBER_REL,
+       RELOPT_DEADREL
 } RelOptKind;
 
 typedef struct RelOptInfo
@@ -344,25 +449,44 @@ typedef struct RelOptInfo
        double          rows;                   /* estimated number of result tuples */
        int                     width;                  /* estimated avg width of result tuples */
 
+       /* per-relation planner control flags */
+       bool            consider_startup;               /* keep cheap-startup-cost paths? */
+       bool            consider_param_startup; /* ditto, for parameterized paths? */
+
        /* materialization information */
        List       *reltargetlist;      /* Vars to be output by scan of relation */
        List       *pathlist;           /* Path structures */
+       List       *ppilist;            /* ParamPathInfos used in pathlist */
        struct Path *cheapest_startup_path;
        struct Path *cheapest_total_path;
        struct Path *cheapest_unique_path;
+       List       *cheapest_parameterized_paths;
 
        /* information about a base rel (not set for join rels!) */
        Index           relid;
+       Oid                     reltablespace;  /* containing tablespace */
        RTEKind         rtekind;                /* RELATION, SUBQUERY, or FUNCTION */
        AttrNumber      min_attr;               /* smallest attrno of rel (often <0) */
        AttrNumber      max_attr;               /* largest attrno of rel */
        Relids     *attr_needed;        /* array indexed [min_attr .. max_attr] */
        int32      *attr_widths;        /* array indexed [min_attr .. max_attr] */
+       List       *lateral_vars;       /* LATERAL Vars and PHVs referenced by rel */
+       Relids          lateral_relids; /* minimum parameterization of rel */
+       Relids          lateral_referencers;    /* rels that reference me laterally */
        List       *indexlist;          /* list of IndexOptInfo */
-       BlockNumber pages;
+       BlockNumber pages;                      /* size estimates derived from pg_class */
        double          tuples;
+       double          allvisfrac;
+       /* use "struct Plan" to avoid including plannodes.h here */
        struct Plan *subplan;           /* if subquery */
-       List       *subrtable;          /* if subquery */
+       PlannerInfo *subroot;           /* if subquery */
+       List       *subplan_params; /* if subquery */
+
+       /* Information about foreign tables and foreign joins */
+       Oid                     serverid;               /* identifies server for the table or join */
+       /* use "struct FdwRoutine" to avoid including fdwapi.h here */
+       struct FdwRoutine *fdwroutine;
+       void       *fdw_private;
 
        /* used by various scans and joins: */
        List       *baserestrictinfo;           /* RestrictInfo structures (if base
@@ -371,65 +495,58 @@ typedef struct RelOptInfo
        List       *joininfo;           /* RestrictInfo structures for join clauses
                                                                 * involving this rel */
        bool            has_eclass_joins;               /* T means joininfo is incomplete */
-
-       /* cached info about inner indexscan paths for relation: */
-       Relids          index_outer_relids;             /* other relids in indexable join
-                                                                                * clauses */
-       List       *index_inner_paths;          /* InnerIndexscanInfo nodes */
-
-       /*
-        * Inner indexscans are not in the main pathlist because they are not
-        * usable except in specific join contexts.  We use the index_inner_paths
-        * list just to avoid recomputing the best inner indexscan repeatedly for
-        * similar outer relations.  See comments for InnerIndexscanInfo.
-        */
 } RelOptInfo;
 
 /*
  * IndexOptInfo
  *             Per-index information for planning/optimization
  *
- *             Prior to Postgres 7.0, RelOptInfo was used to describe both relations
- *             and indexes, but that created confusion without actually doing anything
- *             useful.  So now we have a separate IndexOptInfo struct for indexes.
+ *             indexkeys[], indexcollations[], opfamily[], and opcintype[]
+ *             each have ncolumns entries.
  *
- *             opfamily[], indexkeys[], opcintype[], fwdsortop[], revsortop[],
- *             and nulls_first[] each have ncolumns entries.
- *             Note: for historical reasons, the opfamily array has an extra entry
- *             that is always zero.  Some code scans until it sees a zero entry,
- *             rather than looking at ncolumns.
+ *             sortopfamily[], reverse_sort[], and nulls_first[] likewise have
+ *             ncolumns entries, if the index is ordered; but if it is unordered,
+ *             those pointers are NULL.
  *
  *             Zeroes in the indexkeys[] array indicate index columns that are
  *             expressions; there is one element in indexprs for each such column.
  *
- *             For an unordered index, the sortop arrays contains zeroes.      Note that
- *             fwdsortop[] and nulls_first[] describe the sort ordering of a forward
- *             indexscan; we can also consider a backward indexscan, which will
- *             generate sort order described by revsortop/!nulls_first.
+ *             For an ordered index, reverse_sort[] and nulls_first[] describe the
+ *             sort ordering of a forward indexscan; we can also consider a backward
+ *             indexscan, which will generate the reverse ordering.
  *
  *             The indexprs and indpred expressions have been run through
  *             prepqual.c and eval_const_expressions() for ease of matching to
  *             WHERE clauses. indpred is in implicit-AND form.
+ *
+ *             indextlist is a TargetEntry list representing the index columns.
+ *             It provides an equivalent base-relation Var for each simple column,
+ *             and links to the matching indexprs element for each expression column.
  */
 typedef struct IndexOptInfo
 {
        NodeTag         type;
 
        Oid                     indexoid;               /* OID of the index relation */
+       Oid                     reltablespace;  /* tablespace of index (not table) */
        RelOptInfo *rel;                        /* back-link to index's table */
 
-       /* statistics from pg_class */
+       /* index-size statistics (from pg_class and elsewhere) */
        BlockNumber pages;                      /* number of disk pages in index */
        double          tuples;                 /* number of index tuples in index */
+       int                     tree_height;    /* index tree height, or -1 if unknown */
 
        /* index descriptor information */
        int                     ncolumns;               /* number of columns in index */
-       Oid                *opfamily;           /* OIDs of operator families for columns */
        int                *indexkeys;          /* column numbers of index's keys, or 0 */
+       Oid                *indexcollations;    /* OIDs of collations of index columns */
+       Oid                *opfamily;           /* OIDs of operator families for columns */
        Oid                *opcintype;          /* OIDs of opclass declared input data types */
-       Oid                *fwdsortop;          /* OIDs of sort operators for each column */
-       Oid                *revsortop;          /* OIDs of sort operators for backward scan */
+       Oid                *sortopfamily;       /* OIDs of btree opfamilies, if orderable */
+       bool       *reverse_sort;       /* is sort order descending? */
        bool       *nulls_first;        /* do NULLs come first in the sort order? */
+       bool       *canreturn;          /* which index cols can be returned in an
+                                                                * index-only scan? */
        Oid                     relam;                  /* OID of the access method (in pg_am) */
 
        RegProcedure amcostestimate;    /* OID of the access method's cost fcn */
@@ -437,12 +554,18 @@ typedef struct IndexOptInfo
        List       *indexprs;           /* expressions for non-simple index columns */
        List       *indpred;            /* predicate if a partial index, else NIL */
 
+       List       *indextlist;         /* targetlist representing index columns */
+
        bool            predOK;                 /* true if predicate matches query */
        bool            unique;                 /* true if a unique index */
+       bool            immediate;              /* is uniqueness enforced immediately? */
+       bool            hypothetical;   /* true if index doesn't really exist */
+       bool            amcanorderbyop; /* does AM support order by operator result? */
        bool            amoptionalkey;  /* can query omit key for the first column? */
-       bool            amsearchnulls;  /* can AM search for NULL index entries? */
+       bool            amsearcharray;  /* can AM handle ScalarArrayOpExpr quals? */
+       bool            amsearchnulls;  /* can AM search for NULL/NOT NULL entries? */
        bool            amhasgettuple;  /* does AM have amgettuple interface? */
-       bool            amhasgetbitmap; /* does AM have amgetbitmap interface? */
+       bool            amhasgetbitmap; /* does AM have amgetbitmap interface? */
 } IndexOptInfo;
 
 
@@ -456,16 +579,19 @@ typedef struct IndexOptInfo
  * require merging two existing EquivalenceClasses.  At the end of the qual
  * distribution process, we have sets of values that are known all transitively
  * equal to each other, where "equal" is according to the rules of the btree
- * operator family(s) shown in ec_opfamilies.  (We restrict an EC to contain
- * only equalities whose operators belong to the same set of opfamilies.  This
- * could probably be relaxed, but for now it's not worth the trouble, since
- * nearly all equality operators belong to only one btree opclass anyway.)
+ * operator family(s) shown in ec_opfamilies, as well as the collation shown
+ * by ec_collation.  (We restrict an EC to contain only equalities whose
+ * operators belong to the same set of opfamilies.  This could probably be
+ * relaxed, but for now it's not worth the trouble, since nearly all equality
+ * operators belong to only one btree opclass anyway.  Similarly, we suppose
+ * that all or none of the input datatypes are collatable, so that a single
+ * collation value is sufficient.)
  *
  * We also use EquivalenceClasses as the base structure for PathKeys, letting
  * us represent knowledge about different sort orderings being equivalent.
  * Since every PathKey must reference an EquivalenceClass, we will end up
  * with single-member EquivalenceClasses whenever a sort key expression has
- * not been equivalenced to anything else.     It is also possible that such an
+ * not been equivalenced to anything else.  It is also possible that such an
  * EquivalenceClass will contain a volatile expression ("ORDER BY random()"),
  * which is a case that can't arise otherwise since clauses containing
  * volatile functions are never considered mergejoinable.  We mark such
@@ -478,7 +604,7 @@ typedef struct IndexOptInfo
  * We allow equality clauses appearing below the nullable side of an outer join
  * to form EquivalenceClasses, but these have a slightly different meaning:
  * the included values might be all NULL rather than all the same non-null
- * values.     See src/backend/optimizer/README for more on that point.
+ * values.  See src/backend/optimizer/README for more on that point.
  *
  * NB: if ec_merged isn't NULL, this class has been merged into another, and
  * should be ignored in favor of using the pointed-to class.
@@ -488,10 +614,12 @@ typedef struct EquivalenceClass
        NodeTag         type;
 
        List       *ec_opfamilies;      /* btree operator family OIDs */
+       Oid                     ec_collation;   /* collation, if datatypes are collatable */
        List       *ec_members;         /* list of EquivalenceMembers */
        List       *ec_sources;         /* list of generating RestrictInfos */
        List       *ec_derives;         /* list of derived RestrictInfos */
-       Relids          ec_relids;              /* all relids appearing in ec_members */
+       Relids          ec_relids;              /* all relids appearing in ec_members, except
+                                                                * for child members (see below) */
        bool            ec_has_const;   /* any pseudoconstants in ec_members? */
        bool            ec_has_volatile;        /* the (sole) member is a volatile expr */
        bool            ec_below_outer_join;    /* equivalence applies below an OJ */
@@ -511,15 +639,22 @@ typedef struct EquivalenceClass
  * EquivalenceMember - one member expression of an EquivalenceClass
  *
  * em_is_child signifies that this element was built by transposing a member
- * for an inheritance parent relation to represent the corresponding expression
- * on an inheritance child.  The element should be ignored for all purposes
- * except constructing inner-indexscan paths for the child relation.  (Other
- * types of join are driven from transposed joininfo-list entries.)  Note
- * that the EC's ec_relids field does NOT include the child relation.
+ * for an appendrel parent relation to represent the corresponding expression
+ * for an appendrel child.  These members are used for determining the
+ * pathkeys of scans on the child relation and for explicitly sorting the
+ * child when necessary to build a MergeAppend path for the whole appendrel
+ * tree.  An em_is_child member has no impact on the properties of the EC as a
+ * whole; in particular the EC's ec_relids field does NOT include the child
+ * relation.  An em_is_child member should never be marked em_is_const nor
+ * cause ec_has_const or ec_has_volatile to be set, either.  Thus, em_is_child
+ * members are not really full-fledged members of the EC, but just reflections
+ * or doppelgangers of real members.  Most operations on EquivalenceClasses
+ * should ignore em_is_child members, and those that don't should test
+ * em_relids to make sure they only consider relevant members.
  *
  * em_datatype is usually the same as exprType(em_expr), but can be
  * different when dealing with a binary-compatible opfamily; in particular
- * anyarray_ops would never work without this. Use em_datatype when
+ * anyarray_ops would never work without this.  Use em_datatype when
  * looking up a specific btree operator to work with this expression.
  */
 typedef struct EquivalenceMember
@@ -528,6 +663,7 @@ typedef struct EquivalenceMember
 
        Expr       *em_expr;            /* the expression represented */
        Relids          em_relids;              /* all relids appearing in em_expr */
+       Relids          em_nullable_relids;             /* nullable by lower outer joins */
        bool            em_is_const;    /* expression is pseudoconstant? */
        bool            em_is_child;    /* derived version for a child relation? */
        Oid                     em_datatype;    /* the "nominal type" used by the opfamily */
@@ -541,15 +677,15 @@ typedef struct EquivalenceMember
  * represents the primary sort key, the second the first secondary sort key,
  * etc.  The value being sorted is represented by linking to an
  * EquivalenceClass containing that value and including pk_opfamily among its
- * ec_opfamilies.  This is a convenient method because it makes it trivial
- * to detect equivalent and closely-related orderings. (See optimizer/README
- * for more information.)
+ * ec_opfamilies.  The EquivalenceClass tells which collation to use, too.
+ * This is a convenient method because it makes it trivial to detect
+ * equivalent and closely-related orderings. (See optimizer/README for more
+ * information.)
  *
  * Note: pk_strategy is either BTLessStrategyNumber (for ASC) or
- * BTGreaterStrategyNumber (for DESC). We assume that all ordering-capable
+ * BTGreaterStrategyNumber (for DESC).  We assume that all ordering-capable
  * index types will use btree-compatible strategy numbers.
  */
-
 typedef struct PathKey
 {
        NodeTag         type;
@@ -560,17 +696,55 @@ typedef struct PathKey
        bool            pk_nulls_first; /* do NULLs come before normal values? */
 } PathKey;
 
+
+/*
+ * ParamPathInfo
+ *
+ * All parameterized paths for a given relation with given required outer rels
+ * link to a single ParamPathInfo, which stores common information such as
+ * the estimated rowcount for this parameterization.  We do this partly to
+ * avoid recalculations, but mostly to ensure that the estimated rowcount
+ * is in fact the same for every such path.
+ *
+ * Note: ppi_clauses is only used in ParamPathInfos for base relation paths;
+ * in join cases it's NIL because the set of relevant clauses varies depending
+ * on how the join is formed.  The relevant clauses will appear in each
+ * parameterized join path's joinrestrictinfo list, instead.
+ */
+typedef struct ParamPathInfo
+{
+       NodeTag         type;
+
+       Relids          ppi_req_outer;  /* rels supplying parameters used by path */
+       double          ppi_rows;               /* estimated number of result tuples */
+       List       *ppi_clauses;        /* join clauses available from outer rels */
+} ParamPathInfo;
+
+
 /*
  * Type "Path" is used as-is for sequential-scan paths, as well as some other
  * simple plan types that we don't need any extra information in the path for.
  * For other path types it is the first component of a larger struct.
  *
- * Note: "pathtype" is the NodeTag of the Plan node we could build from this
- * Path.  It is partially redundant with the Path's NodeTag, but allows us
- * to use the same Path type for multiple Plan types where there is no need
- * to distinguish the Plan type during path processing.
+ * "pathtype" is the NodeTag of the Plan node we could build from this Path.
+ * It is partially redundant with the Path's NodeTag, but allows us to use
+ * the same Path type for multiple Plan types when there is no need to
+ * distinguish the Plan type during path processing.
+ *
+ * "param_info", if not NULL, links to a ParamPathInfo that identifies outer
+ * relation(s) that provide parameter values to each scan of this path.
+ * That means this path can only be joined to those rels by means of nestloop
+ * joins with this path on the inside.  Also note that a parameterized path
+ * is responsible for testing all "movable" joinclauses involving this rel
+ * and the specified outer rel(s).
+ *
+ * "rows" is the same as parent->rows in simple paths, but in parameterized
+ * paths and UniquePaths it can be less than parent->rows, reflecting the
+ * fact that we've filtered by extra join conditions or removed duplicates.
+ *
+ * "pathkeys" is a List of PathKey nodes (see above), describing the sort
+ * ordering of the path's output rows.
  */
-
 typedef struct Path
 {
        NodeTag         type;
@@ -578,8 +752,10 @@ typedef struct Path
        NodeTag         pathtype;               /* tag identifying scan/join method */
 
        RelOptInfo *parent;                     /* the relation this path can build */
+       ParamPathInfo *param_info;      /* parameterization info, or NULL if none */
 
-       /* estimated execution costs for path (see costsize.c for more info) */
+       /* estimated size/costs for path (see costsize.c for more info) */
+       double          rows;                   /* estimated number of result tuples */
        Cost            startup_cost;   /* cost expended before fetching any tuples */
        Cost            total_cost;             /* total cost (assuming all tuples fetched) */
 
@@ -587,26 +763,47 @@ typedef struct Path
        /* pathkeys is a List of PathKey nodes; see above */
 } Path;
 
+/* Macro for extracting a path's parameterization relids; beware double eval */
+#define PATH_REQ_OUTER(path)  \
+       ((path)->param_info ? (path)->param_info->ppi_req_outer : (Relids) NULL)
+
 /*----------
  * IndexPath represents an index scan over a single index.
  *
+ * This struct is used for both regular indexscans and index-only scans;
+ * path.pathtype is T_IndexScan or T_IndexOnlyScan to show which is meant.
+ *
  * 'indexinfo' is the index to be scanned.
  *
  * 'indexclauses' is a list of index qualification clauses, with implicit
  * AND semantics across the list.  Each clause is a RestrictInfo node from
- * the query's WHERE or JOIN conditions.
+ * the query's WHERE or JOIN conditions.  An empty list implies a full
+ * index scan.
  *
  * 'indexquals' has the same structure as 'indexclauses', but it contains
- * the actual indexqual conditions that can be used with the index.
+ * the actual index qual conditions that can be used with the index.
  * In simple cases this is identical to 'indexclauses', but when special
  * indexable operators appear in 'indexclauses', they are replaced by the
  * derived indexscannable conditions in 'indexquals'.
  *
- * 'isjoininner' is TRUE if the path is a nestloop inner scan (that is,
- * some of the index conditions are join rather than restriction clauses).
- * Note that the path costs will be calculated differently from a plain
- * indexscan in this case, and in addition there's a special 'rows' value
- * different from the parent RelOptInfo's (see below).
+ * 'indexqualcols' is an integer list of index column numbers (zero-based)
+ * of the same length as 'indexquals', showing which index column each qual
+ * is meant to be used with.  'indexquals' is required to be ordered by
+ * index column, so 'indexqualcols' must form a nondecreasing sequence.
+ * (The order of multiple quals for the same index column is unspecified.)
+ *
+ * 'indexorderbys', if not NIL, is a list of ORDER BY expressions that have
+ * been found to be usable as ordering operators for an amcanorderbyop index.
+ * The list must match the path's pathkeys, ie, one expression per pathkey
+ * in the same order.  These are not RestrictInfos, just bare expressions,
+ * since they generally won't yield booleans.  Also, unlike the case for
+ * quals, it's guaranteed that each expression has the index key on the left
+ * side of the operator.
+ *
+ * 'indexorderbycols' is an integer list of index column numbers (zero-based)
+ * of the same length as 'indexorderbys', showing which index column each
+ * ORDER BY expression is meant to be used with.  (There is no restriction
+ * on which index column each ORDER BY can be used with.)
  *
  * 'indexscandir' is one of:
  *             ForwardScanDirection: forward scan of an ordered index
@@ -619,13 +816,7 @@ typedef struct Path
  * 'indextotalcost' and 'indexselectivity' are saved in the IndexPath so that
  * we need not recompute them when considering using the same index in a
  * bitmap index/heap scan (see BitmapHeapPath).  The costs of the IndexPath
- * itself represent the costs of an IndexScan plan type.
- *
- * 'rows' is the estimated result tuple count for the indexscan.  This
- * is the same as path.parent->rows for a simple indexscan, but it is
- * different for a nestloop inner scan, because the additional indexquals
- * coming from join clauses make the scan more selective than the parent
- * rel's restrict clauses alone would do.
+ * itself represent the costs of an IndexScan or IndexOnlyScan plan type.
  *----------
  */
 typedef struct IndexPath
@@ -634,11 +825,12 @@ typedef struct IndexPath
        IndexOptInfo *indexinfo;
        List       *indexclauses;
        List       *indexquals;
-       bool            isjoininner;
+       List       *indexqualcols;
+       List       *indexorderbys;
+       List       *indexorderbycols;
        ScanDirection indexscandir;
        Cost            indextotalcost;
        Selectivity indexselectivity;
-       double          rows;                   /* estimated number of result tuples */
 } IndexPath;
 
 /*
@@ -650,22 +842,18 @@ typedef struct IndexPath
  *
  * The individual indexscans are represented by IndexPath nodes, and any
  * logic on top of them is represented by a tree of BitmapAndPath and
- * BitmapOrPath nodes. Notice that we can use the same IndexPath node both
- * to represent a regular IndexScan plan, and as the child of a BitmapHeapPath
- * that represents scanning the same index using a BitmapIndexScan.  The
- * startup_cost and total_cost figures of an IndexPath always represent the
- * costs to use it as a regular IndexScan.     The costs of a BitmapIndexScan
- * can be computed using the IndexPath's indextotalcost and indexselectivity.
- *
- * BitmapHeapPaths can be nestloop inner indexscans.  The isjoininner and
- * rows fields serve the same purpose as for plain IndexPaths.
+ * BitmapOrPath nodes.  Notice that we can use the same IndexPath node both
+ * to represent a regular (or index-only) index scan plan, and as the child
+ * of a BitmapHeapPath that represents scanning the same index using a
+ * BitmapIndexScan.  The startup_cost and total_cost figures of an IndexPath
+ * always represent the costs to use it as a regular (or index-only)
+ * IndexScan.  The costs of a BitmapIndexScan can be computed using the
+ * IndexPath's indextotalcost and indexselectivity.
  */
 typedef struct BitmapHeapPath
 {
        Path            path;
        Path       *bitmapqual;         /* IndexPath, BitmapAndPath, BitmapOrPath */
-       bool            isjoininner;    /* T if it's a nestloop inner scan */
-       double          rows;                   /* estimated number of result tuples */
 } BitmapHeapPath;
 
 /*
@@ -707,6 +895,69 @@ typedef struct TidPath
        List       *tidquals;           /* qual(s) involving CTID = something */
 } TidPath;
 
+/*
+ * ForeignPath represents a potential scan of a foreign table
+ *
+ * fdw_private stores FDW private data about the scan.  While fdw_private is
+ * not actually touched by the core code during normal operations, it's
+ * generally a good idea to use a representation that can be dumped by
+ * nodeToString(), so that you can examine the structure during debugging
+ * with tools like pprint().
+ */
+typedef struct ForeignPath
+{
+       Path            path;
+       List       *fdw_private;
+} ForeignPath;
+
+/*
+ * CustomPath represents a table scan done by some out-of-core extension.
+ *
+ * We provide a set of hooks here - which the provider must take care to set
+ * up correctly - to allow extensions to supply their own methods of scanning
+ * a relation.  For example, a provider might provide GPU acceleration, a
+ * cache-based scan, or some other kind of logic we haven't dreamed up yet.
+ *
+ * CustomPaths can be injected into the planning process for a relation by
+ * set_rel_pathlist_hook functions.
+ *
+ * Core code must avoid assuming that the CustomPath is only as large as
+ * the structure declared here; providers are allowed to make it the first
+ * element in a larger structure.  (Since the planner never copies Paths,
+ * this doesn't add any complication.)  However, for consistency with the
+ * FDW case, we provide a "custom_private" field in CustomPath; providers
+ * may prefer to use that rather than define another struct type.
+ */
+struct CustomPath;
+
+#define CUSTOMPATH_SUPPORT_BACKWARD_SCAN       0x0001
+#define CUSTOMPATH_SUPPORT_MARK_RESTORE                0x0002
+
+typedef struct CustomPathMethods
+{
+       const char *CustomName;
+
+       /* Convert Path to a Plan */
+       struct Plan *(*PlanCustomPath) (PlannerInfo *root,
+                                                                                               RelOptInfo *rel,
+                                                                                               struct CustomPath *best_path,
+                                                                                               List *tlist,
+                                                                                               List *clauses,
+                                                                                               List *custom_plans);
+       /* Optional: print additional fields besides "private" */
+       void            (*TextOutCustomPath) (StringInfo str,
+                                                                                         const struct CustomPath *node);
+} CustomPathMethods;
+
+typedef struct CustomPath
+{
+       Path            path;
+       uint32          flags;                  /* mask of CUSTOMPATH_* flags, see above */
+       List       *custom_paths;       /* list of child Path nodes, if any */
+       List       *custom_private;
+       const CustomPathMethods *methods;
+} CustomPath;
+
 /*
  * AppendPath represents an Append plan, ie, successive execution of
  * several member plans.
@@ -725,6 +976,22 @@ typedef struct AppendPath
 #define IS_DUMMY_PATH(p) \
        (IsA((p), AppendPath) && ((AppendPath *) (p))->subpaths == NIL)
 
+/* A relation that's been proven empty will have one path that is dummy */
+#define IS_DUMMY_REL(r) \
+       ((r)->cheapest_total_path != NULL && \
+        IS_DUMMY_PATH((r)->cheapest_total_path))
+
+/*
+ * MergeAppendPath represents a MergeAppend plan, ie, the merging of sorted
+ * results from several member plans to produce similarly-sorted output.
+ */
+typedef struct MergeAppendPath
+{
+       Path            path;
+       List       *subpaths;           /* list of component Paths */
+       double          limit_tuples;   /* hard limit on output tuples, or -1 */
+} MergeAppendPath;
+
 /*
  * ResultPath represents use of a Result plan node to compute a variable-free
  * targetlist with no underlying tables (a "SELECT expressions" query).
@@ -756,7 +1023,7 @@ typedef struct MaterialPath
  *
  * This is unlike the other Path nodes in that it can actually generate
  * different plans: either hash-based or sort-based implementation, or a
- * no-op if the input path can be proven distinct already.     The decision
+ * no-op if the input path can be proven distinct already.  The decision
  * is sufficiently localized that it's not worth having separate Path node
  * types.  (Note: in the no-op case, we could eliminate the UniquePath node
  * entirely and just return the subpath; but it's convenient to have a
@@ -777,9 +1044,21 @@ typedef struct UniquePath
        UniquePathMethod umethod;
        List       *in_operators;       /* equality operators of the IN clause */
        List       *uniq_exprs;         /* expressions to be made unique */
-       double          rows;                   /* estimated number of result tuples */
 } UniquePath;
 
+/*
+ * GatherPath runs several copies of a plan in parallel and collects the
+ * results.  The parallel leader may also execute the plan, unless the
+ * single_copy flag is set.
+ */
+typedef struct GatherPath
+{
+       Path            path;
+       Path       *subpath;            /* path for each worker */
+       int                     num_workers;    /* number of workers sought to help */
+       bool            single_copy;    /* path must not be executed >1x */
+} GatherPath;
+
 /*
  * All join-type paths share these fields.
  */
@@ -796,8 +1075,9 @@ typedef struct JoinPath
        List       *joinrestrictinfo;           /* RestrictInfos to apply to join */
 
        /*
-        * See the notes for RelOptInfo to understand why joinrestrictinfo is
-        * needed in JoinPath, and can't be merged into the parent RelOptInfo.
+        * See the notes for RelOptInfo and ParamPathInfo to understand why
+        * joinrestrictinfo is needed in JoinPath, and can't be merged into the
+        * parent RelOptInfo.
         */
 } JoinPath;
 
@@ -810,6 +1090,14 @@ typedef JoinPath NestPath;
 /*
  * A mergejoin path has these fields.
  *
+ * Unlike other path types, a MergePath node doesn't represent just a single
+ * run-time plan node: it can represent up to four.  Aside from the MergeJoin
+ * node itself, there can be a Sort node for the outer input, a Sort node
+ * for the inner input, and/or a Material node for the inner input.  We could
+ * represent these nodes by separate path nodes, but considering how many
+ * different merge paths are investigated during a complex join problem,
+ * it seems better to avoid unnecessary palloc overhead.
+ *
  * path_mergeclauses lists the clauses (in the form of RestrictInfos)
  * that will be used in the merge.
  *
@@ -821,7 +1109,10 @@ typedef JoinPath NestPath;
  * outersortkeys (resp. innersortkeys) is NIL if the outer path
  * (resp. inner path) is already ordered appropriately for the
  * mergejoin.  If it is not NIL then it is a PathKeys list describing
- * the ordering that must be created by an explicit sort step.
+ * the ordering that must be created by an explicit Sort node.
+ *
+ * materialize_inner is TRUE if a Material node should be placed atop the
+ * inner input.  This may appear with or without an inner Sort step.
  */
 
 typedef struct MergePath
@@ -830,6 +1121,7 @@ typedef struct MergePath
        List       *path_mergeclauses;          /* join clauses to be used for merge */
        List       *outersortkeys;      /* keys for explicit sort, if any */
        List       *innersortkeys;      /* keys for explicit sort, if any */
+       bool            materialize_inner;              /* add Materialize to inner? */
 } MergePath;
 
 /*
@@ -845,6 +1137,7 @@ typedef struct HashPath
 {
        JoinPath        jpath;
        List       *path_hashclauses;           /* join clauses used for hashing */
+       int                     num_batches;    /* number of batches expected */
 } HashPath;
 
 /*
@@ -869,7 +1162,7 @@ typedef struct HashPath
  * When we construct a join rel that includes all the base rels referenced
  * in a multi-relation restriction clause, we place that clause into the
  * joinrestrictinfo lists of paths for the join rel, if neither left nor
- * right sub-path includes all base rels referenced in the clause.     The clause
+ * right sub-path includes all base rels referenced in the clause.  The clause
  * will be applied at that join level, and will not propagate any further up
  * the join tree.  (Note: the "predicate migration" code was once intended to
  * push restriction clauses up and down the plan tree based on evaluation
@@ -909,7 +1202,7 @@ typedef struct HashPath
  * that appeared elsewhere in the tree and were pushed down to the join rel
  * because they used no other rels.  That's what the is_pushed_down flag is
  * for; it tells us that a qual is not an OUTER JOIN qual for the set of base
- * rels listed in required_relids.     A clause that originally came from WHERE
+ * rels listed in required_relids.  A clause that originally came from WHERE
  * or an INNER JOIN condition will *always* have its is_pushed_down flag set.
  * It's possible for an OUTER JOIN clause to be marked is_pushed_down too,
  * if we decide that it can be pushed down into the nullable side of the join.
@@ -919,10 +1212,25 @@ typedef struct HashPath
  *
  * RestrictInfo nodes also contain an outerjoin_delayed flag, which is true
  * if the clause's applicability must be delayed due to any outer joins
- * appearing below its own syntactic level (ie, it references any Vars from
- * the nullable side of any lower outer join).
+ * appearing below it (ie, it has to be postponed to some join level higher
+ * than the set of relations it actually references).
  *
- * In general, the referenced clause might be arbitrarily complex.     The
+ * There is also an outer_relids field, which is NULL except for outer join
+ * clauses; for those, it is the set of relids on the outer side of the
+ * clause's outer join.  (These are rels that the clause cannot be applied to
+ * in parameterized scans, since pushing it into the join's outer side would
+ * lead to wrong answers.)
+ *
+ * There is also a nullable_relids field, which is the set of rels the clause
+ * references that can be forced null by some outer join below the clause.
+ *
+ * outerjoin_delayed = true is subtly different from nullable_relids != NULL:
+ * a clause might reference some nullable rels and yet not be
+ * outerjoin_delayed because it also references all the other rels of the
+ * outer join(s). A clause that is not outerjoin_delayed can be enforced
+ * anywhere it is computable.
+ *
+ * In general, the referenced clause might be arbitrarily complex.  The
  * kinds of clauses we can handle as indexscan quals, mergejoin clauses,
  * or hashjoin clauses are limited (e.g., no volatile functions).  The code
  * for each kind of path is responsible for identifying the restrict clauses
@@ -947,7 +1255,7 @@ typedef struct HashPath
  *
  * The pseudoconstant flag is set true if the clause contains no Vars of
  * the current query level and no volatile functions.  Such a clause can be
- * pulled out and used as a one-time qual in a gating Result node.     We keep
+ * pulled out and used as a one-time qual in a gating Result node.  We keep
  * pseudoconstant clauses in the same lists as other RestrictInfos so that
  * the regular clause-pushing machinery can assign them to the correct join
  * level, but they need to be treated specially for cost and selectivity
@@ -957,7 +1265,7 @@ typedef struct HashPath
  *
  * When join clauses are generated from EquivalenceClasses, there may be
  * several equally valid ways to enforce join equivalence, of which we need
- * apply only one.     We mark clauses of this kind by setting parent_ec to
+ * apply only one.  We mark clauses of this kind by setting parent_ec to
  * point to the generating EquivalenceClass.  Multiple clauses with the same
  * parent_ec in the same join are redundant.
  */
@@ -970,7 +1278,7 @@ typedef struct RestrictInfo
 
        bool            is_pushed_down; /* TRUE if clause was pushed down in level */
 
-       bool            outerjoin_delayed;      /* TRUE if delayed by lower outer join */
+       bool            outerjoin_delayed;              /* TRUE if delayed by lower outer join */
 
        bool            can_join;               /* see comment above */
 
@@ -982,6 +1290,12 @@ typedef struct RestrictInfo
        /* The set of relids required to evaluate the clause: */
        Relids          required_relids;
 
+       /* If an outer-join clause, the outer-side relations, else NULL: */
+       Relids          outer_relids;
+
+       /* The relids used in the clause that are nullable by lower outer joins: */
+       Relids          nullable_relids;
+
        /* These fields are set for any binary opclause: */
        Relids          left_relids;    /* relids in left side of clause */
        Relids          right_relids;   /* relids in right side of clause */
@@ -995,10 +1309,10 @@ typedef struct RestrictInfo
        /* cache space for cost and selectivity */
        QualCost        eval_cost;              /* eval cost of clause; -1 if not yet set */
        Selectivity norm_selec;         /* selectivity for "normal" (JOIN_INNER)
-                                                                * semantics; -1 if not yet set; >1 means
-                                                                * redundant clause */
-       Selectivity outer_selec;        /* selectivity for outer join semantics;
-                                                                * -1 if not yet set */
+                                                                * semantics; -1 if not yet set; >1 means a
+                                                                * redundant clause */
+       Selectivity outer_selec;        /* selectivity for outer join semantics; -1 if
+                                                                * not yet set */
 
        /* valid if clause is mergejoinable, else NIL */
        List       *mergeopfamilies;    /* opfamilies containing clause operator */
@@ -1032,6 +1346,7 @@ typedef struct MergeScanSelCache
 {
        /* Ordering details (cache lookup key) */
        Oid                     opfamily;               /* btree opfamily defining the ordering */
+       Oid                     collation;              /* collation for the ordering */
        int                     strategy;               /* sort direction (ASC or DESC) */
        bool            nulls_first;    /* do NULLs come before normal values? */
        /* Results */
@@ -1041,42 +1356,6 @@ typedef struct MergeScanSelCache
        Selectivity rightendsel;        /* last-join fraction for clause right side */
 } MergeScanSelCache;
 
-/*
- * Inner indexscan info.
- *
- * An inner indexscan is one that uses one or more joinclauses as index
- * conditions (perhaps in addition to plain restriction clauses).  So it
- * can only be used as the inner path of a nestloop join where the outer
- * relation includes all other relids appearing in those joinclauses.
- * The set of usable joinclauses, and thus the best inner indexscan,
- * thus varies depending on which outer relation we consider; so we have
- * to recompute the best such paths for every join.  To avoid lots of
- * redundant computation, we cache the results of such searches.  For
- * each relation we compute the set of possible otherrelids (all relids
- * appearing in joinquals that could become indexquals for this table).
- * Two outer relations whose relids have the same intersection with this
- * set will have the same set of available joinclauses and thus the same
- * best inner indexscans for the inner relation.  By taking the intersection
- * before scanning the cache, we avoid recomputing when considering
- * join rels that differ only by the inclusion of irrelevant other rels.
- *
- * The search key also includes a bool showing whether the join being
- * considered is an outer join.  Since we constrain the join order for
- * outer joins, I believe that this bool can only have one possible value
- * for any particular lookup key; but store it anyway to avoid confusion.
- */
-
-typedef struct InnerIndexscanInfo
-{
-       NodeTag         type;
-       /* The lookup key: */
-       Relids          other_relids;   /* a set of relevant other relids */
-       bool            isouterjoin;    /* true if join is outer */
-       /* Best paths for this lookup key (NULL if no available indexscans): */
-       Path       *cheapest_startup_innerpath;         /* cheapest startup cost */
-       Path       *cheapest_total_innerpath;           /* cheapest total cost */
-} InnerIndexscanInfo;
-
 /*
  * Placeholder node for an expression to be evaluated below the top level
  * of a plan tree.  This is used during planning to represent the contained
@@ -1104,7 +1383,7 @@ typedef struct PlaceHolderVar
  * "Special join" info.
  *
  * One-sided outer joins constrain the order of joining partially but not
- * completely. We flatten such joins into the planner's top-level list of
+ * completely.  We flatten such joins into the planner's top-level list of
  * relations to join, but record information about each outer join in a
  * SpecialJoinInfo struct.  These structs are kept in the PlannerInfo node's
  * join_info_list.
@@ -1134,13 +1413,15 @@ typedef struct PlaceHolderVar
  * to be evaluated after this join is formed (because it references the RHS).
  * Any outer joins that have such a clause and this join in their RHS cannot
  * commute with this join, because that would leave noplace to check the
- * pushed-down clause. (We don't track this for FULL JOINs, either.)
+ * pushed-down clause.  (We don't track this for FULL JOINs, either.)
  *
- * join_quals is an implicit-AND list of the quals syntactically associated
- * with the join (they may or may not end up being applied at the join level).
- * This is just a side list and does not drive actual application of quals.
- * For JOIN_SEMI joins, this is cleared to NIL in create_unique_path() if
- * the join is found not to be suitable for a uniqueify-the-RHS plan.
+ * For a semijoin, we also extract the join operators and their RHS arguments
+ * and set semi_operators, semi_rhs_exprs, semi_can_btree, and semi_can_hash.
+ * This is done in support of possibly unique-ifying the RHS, so we don't
+ * bother unless at least one of semi_can_btree and semi_can_hash can be set
+ * true.  (You might expect that this information would be computed during
+ * join planning; but it's helpful to have it available during planning of
+ * parameterized table scans, so we store it in the SpecialJoinInfo structs.)
  *
  * jointype is never JOIN_RIGHT; a RIGHT JOIN is handled by switching
  * the inputs to make it a LEFT JOIN.  So the allowed values of jointype
@@ -1153,7 +1434,7 @@ typedef struct PlaceHolderVar
  * SpecialJoinInfos with jointype == JOIN_INNER for outer joins, since for
  * cost estimation purposes it is sometimes useful to know the join size under
  * plain innerjoin semantics.  Note that lhs_strict, delay_upper_joins, and
- * join_quals are not set meaningfully within such structs.
+ * of course the semi_xxx fields are not set meaningfully within such structs.
  */
 
 typedef struct SpecialJoinInfo
@@ -1166,9 +1447,50 @@ typedef struct SpecialJoinInfo
        JoinType        jointype;               /* always INNER, LEFT, FULL, SEMI, or ANTI */
        bool            lhs_strict;             /* joinclause is strict for some LHS rel */
        bool            delay_upper_joins;              /* can't commute with upper RHS */
-       List       *join_quals;         /* join quals, in implicit-AND list format */
+       /* Remaining fields are set only for JOIN_SEMI jointype: */
+       bool            semi_can_btree; /* true if semi_operators are all btree */
+       bool            semi_can_hash;  /* true if semi_operators are all hash */
+       List       *semi_operators; /* OIDs of equality join operators */
+       List       *semi_rhs_exprs; /* righthand-side expressions of these ops */
 } SpecialJoinInfo;
 
+/*
+ * "Lateral join" info.
+ *
+ * Lateral references constrain the join order in a way that's somewhat like
+ * outer joins, though different in detail.  We construct a LateralJoinInfo
+ * for each lateral cross-reference, placing them in the PlannerInfo node's
+ * lateral_info_list.
+ *
+ * For unflattened LATERAL RTEs, we generate LateralJoinInfo(s) in which
+ * lateral_rhs is the relid of the LATERAL baserel, and lateral_lhs is a set
+ * of relids of baserels it references, all of which must be present on the
+ * LHS to compute a parameter needed by the RHS.  Typically, lateral_lhs is
+ * a singleton, but it can include multiple rels if the RHS references a
+ * PlaceHolderVar with a multi-rel ph_eval_at level.  We disallow joining to
+ * only part of the LHS in such cases, since that would result in a join tree
+ * with no convenient place to compute the PHV.
+ *
+ * When an appendrel contains lateral references (eg "LATERAL (SELECT x.col1
+ * UNION ALL SELECT y.col2)"), the LateralJoinInfos reference the parent
+ * baserel not the member otherrels, since it is the parent relid that is
+ * considered for joining purposes.
+ *
+ * If any LATERAL RTEs were flattened into the parent query, it is possible
+ * that the query now contains PlaceHolderVars containing lateral references,
+ * representing expressions that need to be evaluated at particular spots in
+ * the jointree but contain lateral references to Vars from elsewhere.  These
+ * give rise to LateralJoinInfos in which lateral_rhs is the evaluation point
+ * of a PlaceHolderVar and lateral_lhs is the set of lateral rels it needs.
+ */
+
+typedef struct LateralJoinInfo
+{
+       NodeTag         type;
+       Relids          lateral_lhs;    /* rels needed to compute a lateral value */
+       Relids          lateral_rhs;    /* rel where lateral value is needed */
+} LateralJoinInfo;
+
 /*
  * Append-relation info.
  *
@@ -1219,7 +1541,7 @@ typedef struct AppendRelInfo
        /*
         * For an inheritance appendrel, the parent and child are both regular
         * relations, and we store their rowtype OIDs here for use in translating
-        * whole-row Vars.      For a UNION-ALL appendrel, the parent and child are
+        * whole-row Vars.  For a UNION-ALL appendrel, the parent and child are
         * both subqueries with no named rowtype, and we store InvalidOid here.
         */
        Oid                     parent_reltype; /* OID of parent's composite type */
@@ -1238,7 +1560,7 @@ typedef struct AppendRelInfo
         * Vars are special-cased, and system columns (attno < 0) need no special
         * translation since their attnos are the same for all tables.
         *
-        * Caution: the Vars have varlevelsup = 0.      Be careful to adjust as needed
+        * Caution: the Vars have varlevelsup = 0.  Be careful to adjust as needed
         * when copying into a subquery.
         */
        List       *translated_vars;    /* Expressions in the child's Vars */
@@ -1264,8 +1586,17 @@ typedef struct AppendRelInfo
  * then allow it to bubble up like a Var until the ph_needed join level.
  * ph_needed has the same definition as attr_needed for a regular Var.
  *
+ * The PlaceHolderVar's expression might contain LATERAL references to vars
+ * coming from outside its syntactic scope.  If so, those rels are *not*
+ * included in ph_eval_at, but they are recorded in ph_lateral.
+ *
+ * Notice that when ph_eval_at is a join rather than a single baserel, the
+ * PlaceHolderInfo may create constraints on join order: the ph_eval_at join
+ * has to be formed below any outer joins that should null the PlaceHolderVar.
+ *
  * We create a PlaceHolderInfo only after determining that the PlaceHolderVar
- * is actually referenced in the plan tree.
+ * is actually referenced in the plan tree, so that unreferenced placeholders
+ * don't result in unnecessary constraints on join order.
  */
 
 typedef struct PlaceHolderInfo
@@ -1275,43 +1606,161 @@ typedef struct PlaceHolderInfo
        Index           phid;                   /* ID for PH (unique within planner run) */
        PlaceHolderVar *ph_var;         /* copy of PlaceHolderVar tree */
        Relids          ph_eval_at;             /* lowest level we can evaluate value at */
+       Relids          ph_lateral;             /* relids of contained lateral refs, if any */
        Relids          ph_needed;              /* highest level the value is needed at */
        int32           ph_width;               /* estimated attribute width */
 } PlaceHolderInfo;
 
 /*
- * glob->paramlist keeps track of the PARAM_EXEC slots that we have decided
- * we need for the query.  At runtime these slots are used to pass values
- * either down into subqueries (for outer references in subqueries) or up out
- * of subqueries (for the results of a subplan).  The n'th entry in the list
- * (n counts from 0) corresponds to Param->paramid = n.
- *
- * Each paramlist item shows the absolute query level it is associated with,
- * where the outermost query is level 1 and nested subqueries have higher
- * numbers.  The item the parameter slot represents can be one of three kinds:
- *
- * A Var: the slot represents a variable of that level that must be passed
- * down because subqueries have outer references to it.  The varlevelsup
- * value in the Var will always be zero.
+ * For each potentially index-optimizable MIN/MAX aggregate function,
+ * root->minmax_aggs stores a MinMaxAggInfo describing it.
+ */
+typedef struct MinMaxAggInfo
+{
+       NodeTag         type;
+
+       Oid                     aggfnoid;               /* pg_proc Oid of the aggregate */
+       Oid                     aggsortop;              /* Oid of its sort operator */
+       Expr       *target;                     /* expression we are aggregating on */
+       PlannerInfo *subroot;           /* modified "root" for planning the subquery */
+       Path       *path;                       /* access path for subquery */
+       Cost            pathcost;               /* estimated cost to fetch first row */
+       Param      *param;                      /* param for subplan's output */
+} MinMaxAggInfo;
+
+/*
+ * At runtime, PARAM_EXEC slots are used to pass values around from one plan
+ * node to another.  They can be used to pass values down into subqueries (for
+ * outer references in subqueries), or up out of subqueries (for the results
+ * of a subplan), or from a NestLoop plan node into its inner relation (when
+ * the inner scan is parameterized with values from the outer relation).
+ * The planner is responsible for assigning nonconflicting PARAM_EXEC IDs to
+ * the PARAM_EXEC Params it generates.
+ *
+ * Outer references are managed via root->plan_params, which is a list of
+ * PlannerParamItems.  While planning a subquery, each parent query level's
+ * plan_params contains the values required from it by the current subquery.
+ * During create_plan(), we use plan_params to track values that must be
+ * passed from outer to inner sides of NestLoop plan nodes.
+ *
+ * The item a PlannerParamItem represents can be one of three kinds:
+ *
+ * A Var: the slot represents a variable of this level that must be passed
+ * down because subqueries have outer references to it, or must be passed
+ * from a NestLoop node to its inner scan.  The varlevelsup value in the Var
+ * will always be zero.
+ *
+ * A PlaceHolderVar: this works much like the Var case, except that the
+ * entry is a PlaceHolderVar node with a contained expression.  The PHV
+ * will have phlevelsup = 0, and the contained expression is adjusted
+ * to match in level.
  *
  * An Aggref (with an expression tree representing its argument): the slot
  * represents an aggregate expression that is an outer reference for some
  * subquery.  The Aggref itself has agglevelsup = 0, and its argument tree
  * is adjusted to match in level.
  *
- * A Param: the slot holds the result of a subplan (it is a setParam item
- * for that subplan).  The absolute level shown for such items corresponds
- * to the parent query of the subplan.
- *
- * Note: we detect duplicate Var parameters and coalesce them into one slot,
- * but we do not do this for Aggref or Param slots.
+ * Note: we detect duplicate Var and PlaceHolderVar parameters and coalesce
+ * them into one slot, but we do not bother to do that for Aggrefs.
+ * The scope of duplicate-elimination only extends across the set of
+ * parameters passed from one query level into a single subquery, or for
+ * nestloop parameters across the set of nestloop parameters used in a single
+ * query level.  So there is no possibility of a PARAM_EXEC slot being used
+ * for conflicting purposes.
+ *
+ * In addition, PARAM_EXEC slots are assigned for Params representing outputs
+ * from subplans (values that are setParam items for those subplans).  These
+ * IDs need not be tracked via PlannerParamItems, since we do not need any
+ * duplicate-elimination nor later processing of the represented expressions.
+ * Instead, we just record the assignment of the slot number by incrementing
+ * root->glob->nParamExec.
  */
 typedef struct PlannerParamItem
 {
        NodeTag         type;
 
-       Node       *item;                       /* the Var, Aggref, or Param */
-       Index           abslevel;               /* its absolute query level */
+       Node       *item;                       /* the Var, PlaceHolderVar, or Aggref */
+       int                     paramId;                /* its assigned PARAM_EXEC slot number */
 } PlannerParamItem;
 
+/*
+ * When making cost estimates for a SEMI or ANTI join, there are some
+ * correction factors that are needed in both nestloop and hash joins
+ * to account for the fact that the executor can stop scanning inner rows
+ * as soon as it finds a match to the current outer row.  These numbers
+ * depend only on the selected outer and inner join relations, not on the
+ * particular paths used for them, so it's worthwhile to calculate them
+ * just once per relation pair not once per considered path.  This struct
+ * is filled by compute_semi_anti_join_factors and must be passed along
+ * to the join cost estimation functions.
+ *
+ * outer_match_frac is the fraction of the outer tuples that are
+ *             expected to have at least one match.
+ * match_count is the average number of matches expected for
+ *             outer tuples that have at least one match.
+ */
+typedef struct SemiAntiJoinFactors
+{
+       Selectivity outer_match_frac;
+       Selectivity match_count;
+} SemiAntiJoinFactors;
+
+/*
+ * Struct for extra information passed to subroutines of add_paths_to_joinrel
+ *
+ * restrictlist contains all of the RestrictInfo nodes for restriction
+ *             clauses that apply to this join
+ * mergeclause_list is a list of RestrictInfo nodes for available
+ *             mergejoin clauses in this join
+ * sjinfo is extra info about special joins for selectivity estimation
+ * semifactors is as shown above (only valid for SEMI or ANTI joins)
+ * param_source_rels are OK targets for parameterization of result paths
+ * extra_lateral_rels are additional parameterization for result paths
+ */
+typedef struct JoinPathExtraData
+{
+       List       *restrictlist;
+       List       *mergeclause_list;
+       SpecialJoinInfo *sjinfo;
+       SemiAntiJoinFactors semifactors;
+       Relids          param_source_rels;
+       Relids          extra_lateral_rels;
+} JoinPathExtraData;
+
+/*
+ * For speed reasons, cost estimation for join paths is performed in two
+ * phases: the first phase tries to quickly derive a lower bound for the
+ * join cost, and then we check if that's sufficient to reject the path.
+ * If not, we come back for a more refined cost estimate.  The first phase
+ * fills a JoinCostWorkspace struct with its preliminary cost estimates
+ * and possibly additional intermediate values.  The second phase takes
+ * these values as inputs to avoid repeating work.
+ *
+ * (Ideally we'd declare this in cost.h, but it's also needed in pathnode.h,
+ * so seems best to put it here.)
+ */
+typedef struct JoinCostWorkspace
+{
+       /* Preliminary cost estimates --- must not be larger than final ones! */
+       Cost            startup_cost;   /* cost expended before fetching any tuples */
+       Cost            total_cost;             /* total cost (assuming all tuples fetched) */
+
+       /* Fields below here should be treated as private to costsize.c */
+       Cost            run_cost;               /* non-startup cost components */
+
+       /* private for cost_nestloop code */
+       Cost            inner_run_cost; /* also used by cost_mergejoin code */
+       Cost            inner_rescan_run_cost;
+
+       /* private for cost_mergejoin code */
+       double          outer_rows;
+       double          inner_rows;
+       double          outer_skip_rows;
+       double          inner_skip_rows;
+
+       /* private for cost_hashjoin code */
+       int                     numbuckets;
+       int                     numbatches;
+} JoinCostWorkspace;
+
 #endif   /* RELATION_H */