From 23f29fd7b63fffa0e1fc2c65c5883f262591d848 Mon Sep 17 00:00:00 2001
From: amit <amitlangote09@gmail.com>
Date: Wed, 27 Jul 2016 16:59:21 +0900
Subject: [PATCH 7/8] Tuple routing for partitioned tables.

Both COPY FROM and INSERT.
---
 src/backend/catalog/partition.c         |  317 ++++++++++++++++++++++++++++++-
 src/backend/commands/copy.c             |  222 +++++++++++++++++++++-
 src/backend/commands/tablecmds.c        |    1 +
 src/backend/executor/execMain.c         |   59 ++++++-
 src/backend/executor/nodeModifyTable.c  |  160 ++++++++++++++++
 src/backend/nodes/copyfuncs.c           |    1 +
 src/backend/nodes/outfuncs.c            |    1 +
 src/backend/nodes/readfuncs.c           |    1 +
 src/backend/optimizer/plan/createplan.c |   76 ++++++++
 src/backend/parser/analyze.c            |    8 +
 src/include/catalog/partition.h         |    7 +
 src/include/executor/executor.h         |    6 +
 src/include/nodes/execnodes.h           |   10 +
 src/include/nodes/plannodes.h           |    1 +
 src/test/regress/expected/insert.out    |   52 +++++
 src/test/regress/sql/insert.sql         |   25 +++
 16 files changed, 940 insertions(+), 7 deletions(-)

diff --git a/src/backend/catalog/partition.c b/src/backend/catalog/partition.c
index 7bc4b15..4e253c1 100644
--- a/src/backend/catalog/partition.c
+++ b/src/backend/catalog/partition.c
@@ -185,6 +185,18 @@ static List *generate_partition_qual(Relation rel, bool recurse);
 static PartitionTreeNode GetPartitionTreeNodeRecurse(Relation rel, int offset, int lockmode);
 static int get_leaf_partition_count(PartitionTreeNode parent);
 
+/* Support get_partition_for_tuple() */
+static PartitionKeyExecInfo *BuildPartitionKeyExecInfo(Relation rel);
+static void FormPartitionKeyDatum(PartitionKeyExecInfo *pkinfo,
+							TupleTableSlot *slot,
+							EState *estate,
+							Datum *values,
+							bool *isnull);
+static int list_partition_for_tuple(PartitionKey key, PartitionDesc pdesc,
+							Datum value, bool isnull);
+static int range_partition_for_tuple(PartitionKey key, PartitionDesc pdesc,
+							Datum *tuple);
+
 /* List partition related support functions */
 static bool equal_list_info(PartitionKey key,
 				PartitionListInfo *l1, PartitionListInfo *l2);
@@ -198,6 +210,8 @@ static PartitionRangeBound *copy_range_bound(PartitionKey key, PartitionRangeBou
 static bool equal_range_info(PartitionKey key,
 				 PartitionRangeInfo *r1, PartitionRangeInfo *r2);
 static int32 partition_rbound_cmp(PartitionKey key, PartitionRangeBound *b1, void *arg);
+static int32 partition_rbound_datum_cmp(PartitionKey key, PartitionRangeBound *bound,
+						   void *arg);
 static bool partition_rbound_eq(PartitionKey key,
 					PartitionRangeBound *b1, PartitionRangeBound *b2);
 typedef int32 (*partition_rbound_bsearch_cmp_fn) (PartitionKey,
@@ -1443,7 +1457,7 @@ GetPartitionTreeNodeRecurse(Relation rel, int offset, int lockmode)
 
 	/* First build our own node */
 	parent = (PartitionTreeNode) palloc0(sizeof(PartitionTreeNodeData));
-	parent->pkinfo = NULL;
+	parent->pkinfo = BuildPartitionKeyExecInfo(rel);
 	parent->pdesc = RelationGetPartitionDesc(rel);
 	parent->relid = RelationGetRelid(rel);
 	parent->offset = offset;
@@ -1527,6 +1541,273 @@ get_leaf_partition_count(PartitionTreeNode parent)
 	return result;
 }
 
+/*
+ *	BuildPartitionKeyExecInfo
+ *		Construct a list of PartitionKeyExecInfo records for an open
+ *		relation
+ *
+ * PartitionKeyExecInfo stores the information about the partition key
+ * that's needed when inserting tuples into a partitioned table; especially,
+ * partition key expression state if there are any expression columns in
+ * the partition key.  Normally we build a PartitionKeyExecInfo for a
+ * partitioned table just once per command, and then use it for (potentially)
+ * many tuples.
+ *
+ */
+static PartitionKeyExecInfo *
+BuildPartitionKeyExecInfo(Relation rel)
+{
+	PartitionKeyExecInfo   *pkinfo;
+
+	pkinfo = (PartitionKeyExecInfo *) palloc0(sizeof(PartitionKeyExecInfo));
+	pkinfo->pi_Key = RelationGetPartitionKey(rel);
+	pkinfo->pi_ExpressionState = NIL;
+
+	return pkinfo;
+}
+
+/* ----------------
+ *		FormPartitionKeyDatum
+ *			Construct values[] and isnull[] arrays for the partition key
+ *			of a tuple.
+ *
+ *	pkinfo			partition key execution info
+ *	slot			Heap tuple from which to extract partition key
+ *	estate			executor state for evaluating any partition key
+ *					expressions (must be non-NULL)
+ *	values			Array of partition key Datums (output area)
+ *	isnull			Array of is-null indicators (output area)
+ *
+ * the ecxt_scantuple slot of estate's per-tuple expr context must point to
+ * the heap tuple passed in.
+ * ----------------
+ */
+static void
+FormPartitionKeyDatum(PartitionKeyExecInfo *pkinfo,
+					  TupleTableSlot *slot,
+					  EState *estate,
+					  Datum *values,
+					  bool *isnull)
+{
+	ListCell   *partexpr_item;
+	int			i;
+
+	if (pkinfo->pi_Key->partexprs != NIL && pkinfo->pi_ExpressionState == NIL)
+	{
+		/* Check caller has set up context correctly */
+		Assert(estate != NULL &&
+			   GetPerTupleExprContext(estate)->ecxt_scantuple == slot);
+
+		/* First time through, set up expression evaluation state */
+		pkinfo->pi_ExpressionState = (List *)
+			ExecPrepareExpr((Expr *) pkinfo->pi_Key->partexprs,
+							estate);
+	}
+
+	partexpr_item = list_head(pkinfo->pi_ExpressionState);
+	for (i = 0; i < pkinfo->pi_Key->partnatts; i++)
+	{
+		AttrNumber	keycol = pkinfo->pi_Key->partattrs[i];
+		Datum		pkDatum;
+		bool		isNull;
+
+		if (keycol != 0)
+		{
+			/* Plain column; get the value directly from the heap tuple */
+			pkDatum = slot_getattr(slot, keycol, &isNull);
+		}
+		else
+		{
+			/* Expression; need to evaluate it */
+			if (partexpr_item == NULL)
+				elog(ERROR, "wrong number of partition key expressions");
+			pkDatum = ExecEvalExprSwitchContext((ExprState *) lfirst(partexpr_item),
+											   GetPerTupleExprContext(estate),
+											   &isNull,
+											   NULL);
+			partexpr_item = lnext(partexpr_item);
+		}
+		values[i] = pkDatum;
+		isnull[i] = isNull;
+	}
+
+	if (partexpr_item != NULL)
+		elog(ERROR, "wrong number of partition key expressions");
+}
+
+/*
+ * get_partition_for_tuple
+ *		Recursively finds the "leaf" partition for tuple
+ *
+ * parent is the root of the partition tree through which we are trying
+ * to route the tuple contained in *slot.
+ *
+ * Returned value is the sequence number of the leaf partition thus found,
+ * or -1 if no leaf partition is found for the tuple.  *failed_at is set
+ * to the OID of the partitioned table whose partition was not found in
+ * the latter case.
+ */
+int
+get_partition_for_tuple(PartitionTreeNode parent,
+						TupleTableSlot *slot,
+						EState *estate,
+						Oid *failed_at)
+{
+	PartitionKeyExecInfo   *pkinfo = parent->pkinfo;
+	PartitionTreeNode		node,
+							prev;
+	Datum	values[PARTITION_MAX_KEYS];
+	bool	isnull[PARTITION_MAX_KEYS];
+	int		i;
+	int		cur_idx;
+
+	/* Guard against stack overflow due to overly deep partition tree */
+	check_stack_depth();
+
+	if (parent->pdesc->nparts == 0)
+	{
+		*failed_at = parent->relid;
+		return -1;
+	}
+
+	/* Extract partition key from tuple */
+	FormPartitionKeyDatum(pkinfo, slot, estate, values, isnull);
+
+	switch (pkinfo->pi_Key->strategy)
+	{
+		case PARTITION_STRATEGY_LIST:
+			cur_idx = list_partition_for_tuple(pkinfo->pi_Key, parent->pdesc,
+											   values[0], isnull[0]);
+			break;
+
+		case PARTITION_STRATEGY_RANGE:
+			/* Disallow nulls in the partition key of the tuple */
+			for (i = 0; i < pkinfo->pi_Key->partnatts; i++)
+				if (isnull[i])
+					ereport(ERROR,
+					(errcode(ERRCODE_NULL_VALUE_NOT_ALLOWED),
+					 errmsg("range partition key of row contains null")));
+
+			cur_idx = range_partition_for_tuple(pkinfo->pi_Key, parent->pdesc,
+												values);
+			break;
+	}
+
+	/* No partition found at this level */
+	if (cur_idx < 0)
+	{
+		*failed_at = parent->relid;
+		return cur_idx;
+	}
+
+	/*
+	 * The partition we found may either be a leaf partition or partitioned
+	 * itself.  In the latter case, we need to recurse by finding the proper
+	 * node (ie, such that node->index == cur_idx) to pass down.
+	 */
+	prev = parent;
+	node = parent->downlink;
+	while (node != NULL)
+	{
+		if (node->index >= cur_idx)
+			break;
+
+		prev = node;
+		node = node->next;
+	}
+
+	/*
+	 * If we couldn't find a node, that means cur_idx is actually a leaf
+	 * partition.  In the simpler case where all of the left siblings are leaf
+	 * partitions themselves, we can get the correct index to return by adding
+	 * cur_idx to the parent node's offset.  Otherwise, we need to compensate
+	 * for the leaf partitions in the partition subtrees of all the left
+	 * siblings that are partitioned, which, fortunately it suffices to look
+	 * at the rightmost such node.
+	 */
+	Assert(prev != NULL);
+	if (!node || cur_idx < node->index)
+		return prev == parent ? prev->offset + cur_idx
+							  : prev->offset + prev->num_leaf_parts +
+								(cur_idx - prev->index - 1);
+
+	/*
+	 * Must recurse because it turns out that the partition at cur_idx is
+	 * partitioned itself
+	 */
+	Assert (node != NULL && node->index == cur_idx);
+
+	return get_partition_for_tuple(node, slot, estate, failed_at);
+}
+
+/*
+ * list_partition_for_tuple
+ *		Find the list partition for a tuple (arg 'value' contains the
+ *		list partition key of the original tuple)
+ *
+ * Returns -1 if none found.
+ */
+static int
+list_partition_for_tuple(PartitionKey key, PartitionDesc pdesc,
+						 Datum value, bool isnull)
+{
+	PartitionListInfo	listinfo;
+	int			found;
+
+	Assert(pdesc->nparts > 0);
+	Assert(pdesc->boundinfo->strategy == PARTITION_STRATEGY_LIST);
+	listinfo = pdesc->boundinfo->bounds.lists;
+
+	if (isnull && listinfo.has_null)
+		return listinfo.null_index;
+	else if (!isnull)
+	{
+		found = partition_list_values_bsearch(key,
+											  listinfo.values,
+											  listinfo.nvalues,
+											  value);
+		if (found >= 0)
+			return listinfo.indexes[found];
+	}
+
+	/* Control reaches here if isnull and !listinfo->has_null */
+	return -1;
+}
+
+/*
+ * range_partition_for_tuple
+ *		Get the index of the range partition for a tuple (arg 'tuple'
+ *		actually contains the range partition key of the original
+ *		tuple)
+ *
+ * Returns -1 if none found.
+ */
+static int
+range_partition_for_tuple(PartitionKey key, PartitionDesc pdesc, Datum *tuple)
+{
+	int			offset;
+	PartitionRangeInfo	rangeinfo;
+
+	Assert(pdesc->nparts > 0);
+	Assert(pdesc->boundinfo->strategy == PARTITION_STRATEGY_RANGE);
+	rangeinfo = pdesc->boundinfo->bounds.ranges;
+
+	offset = partition_rbound_bsearch(key,
+									  rangeinfo.bounds, rangeinfo.nbounds,
+									  tuple, partition_rbound_datum_cmp,
+									  true, NULL);
+
+	/*
+	 * Offset returned is such that the bound at offset is found to be less
+	 * or equal with the tuple.  That is, the tuple belongs to the partition
+	 * with the rangeinfo.bounds[offset] as the lower bound and
+	 * rangeinfo.bounds[offset+1] as the upper bound, provided the latter is
+	 * indeed an upper (!lower) bound.  If it turns out otherwise, the
+	 * corresponding index will be -1, which means no valid partition exists.
+	 */
+	return rangeinfo.indexes[offset+1];
+}
+
 /* List partition related support functions */
 
 /*
@@ -1772,6 +2053,40 @@ partition_rbound_cmp(PartitionKey key, PartitionRangeBound *b1, void *arg)
 }
 
 /*
+ * Return whether bound <=, =, >= partition key of tuple
+ *
+ * The 3rd argument is void * so that it can be used with
+ * partition_rbound_bsearch()
+ */
+static int32
+partition_rbound_datum_cmp(PartitionKey key, PartitionRangeBound *bound,
+						   void *arg)
+{
+	Datum  *datums1 = bound->datums,
+		   *datums2 = (Datum *) arg;
+	int		i;
+	int32	cmpval;
+
+	for (i = 0; i < key->partnatts; i++)
+	{
+		if (bound->infinite[i])
+			return bound->lower ? -1 : 1;
+
+		cmpval = DatumGetInt32(FunctionCall2Coll(&key->partsupfunc[i],
+												 key->partcollation[i],
+												 datums1[i], datums2[i]));
+		if (cmpval != 0)
+			break;
+	}
+
+	/* If datums are equal and this is an upper bound, tuple > bound */
+	if (cmpval == 0 && !bound->lower)
+		return -1;
+
+	return cmpval;
+}
+
+/*
  * Return whether two range bounds are equal simply by comparing datums
  */
 static bool
diff --git a/src/backend/commands/copy.c b/src/backend/commands/copy.c
index 7a2bf94..cb0ca0e 100644
--- a/src/backend/commands/copy.c
+++ b/src/backend/commands/copy.c
@@ -30,6 +30,7 @@
 #include "commands/defrem.h"
 #include "commands/trigger.h"
 #include "executor/executor.h"
+#include "foreign/fdwapi.h"
 #include "libpq/libpq.h"
 #include "libpq/pqformat.h"
 #include "mb/pg_wchar.h"
@@ -161,6 +162,11 @@ typedef struct CopyStateData
 	ExprState **defexprs;		/* array of default att expressions */
 	bool		volatile_defexprs;		/* is any of defexprs volatile? */
 	List	   *range_table;
+	PartitionTreeNode		partition_tree;	/* partition node tree */
+	ResultRelInfo		   *partitions;
+	TupleConversionMap	  **partition_tupconv_maps;
+	List				   *partition_fdw_priv_lists;
+	int						num_partitions;
 
 	/*
 	 * These variables are used to reduce overhead in textual COPY FROM.
@@ -1397,6 +1403,91 @@ BeginCopy(ParseState *pstate,
 					(errcode(ERRCODE_UNDEFINED_COLUMN),
 					 errmsg("table \"%s\" does not have OIDs",
 							RelationGetRelationName(cstate->rel))));
+
+		/*
+		 * Initialize state for CopyFrom tuple routing.  Watch out for
+		 * any foreign partitions.
+		 */
+		if (is_from && rel->rd_rel->relkind == RELKIND_PARTITIONED_TABLE)
+		{
+			List		   *leaf_parts;
+			ListCell	   *cell;
+			int				i;
+			int				num_leaf_parts;
+			ResultRelInfo  *leaf_part_rri;
+			PlannerInfo *root = makeNode(PlannerInfo);	/* mostly dummy */
+			ModifyTable *plan = makeNode(ModifyTable);	/* ditto */
+			RangeTblEntry *fdw_rte = makeNode(RangeTblEntry);	/* ditto */
+			List		*fdw_private_lists = NIL;
+
+			/* Form the partition node tree and lock partitions */
+			cstate->partition_tree = RelationGetPartitionTreeNode(rel,
+														  RowExclusiveLock);
+			leaf_parts = get_leaf_partition_oids(cstate->partition_tree);
+			num_leaf_parts = list_length(leaf_parts);
+
+			cstate->num_partitions = num_leaf_parts;
+			cstate->partitions = (ResultRelInfo *)
+								palloc0(num_leaf_parts * sizeof(ResultRelInfo));
+			cstate->partition_tupconv_maps = (TupleConversionMap **)
+						palloc0(num_leaf_parts * sizeof(TupleConversionMap *));
+
+			/* For use below, iff a partition found to be a foreign table */
+			plan->operation = CMD_INSERT;
+			plan->plans = list_make1(makeNode(Result));
+			fdw_rte->rtekind = RTE_RELATION;
+			fdw_rte->relkind = RELKIND_FOREIGN_TABLE;
+			root->parse = makeNode(Query);
+			root->parse->rtable = list_make1(fdw_rte);
+
+			leaf_part_rri = cstate->partitions;
+			i = 0;
+			foreach(cell, leaf_parts)
+			{
+				Relation	part_rel;
+
+				part_rel = heap_open(lfirst_oid(cell), RowExclusiveLock);
+
+				/*
+				 * Verify result relation is a valid target for the current
+				 * operation.
+				 */
+				CheckValidResultRel(part_rel, CMD_INSERT);
+
+				InitResultRelInfo(leaf_part_rri,
+								  part_rel,
+								  1,		/* dummy */
+								  false,	/* no need for partition check */
+								  0);
+
+				/* Open partition indices */
+				ExecOpenIndices(leaf_part_rri, false);
+
+				/* Special dance for foreign tables */
+				if (leaf_part_rri->ri_FdwRoutine)
+				{
+					List		  *fdw_private;
+
+					fdw_rte->relid = RelationGetRelid(part_rel);
+					fdw_private = leaf_part_rri->ri_FdwRoutine->PlanForeignModify(root,
+																		  plan,
+																		  1,
+																		  0);
+					fdw_private_lists = lappend(fdw_private_lists, fdw_private);
+				}
+
+				if (!equalTupleDescs(tupDesc, RelationGetDescr(part_rel)))
+					cstate->partition_tupconv_maps[i] =
+								convert_tuples_by_name(tupDesc,
+									RelationGetDescr(part_rel),
+									gettext_noop("could not convert row type"));
+
+				leaf_part_rri++;
+				i++;
+			}
+
+			cstate->partition_fdw_priv_lists = fdw_private_lists;
+		}
 	}
 	else
 	{
@@ -2255,6 +2346,7 @@ CopyFrom(CopyState cstate)
 	Datum	   *values;
 	bool	   *nulls;
 	ResultRelInfo *resultRelInfo;
+	ResultRelInfo *saved_resultRelInfo = NULL;
 	EState	   *estate = CreateExecutorState(); /* for ExecConstraints() */
 	ExprContext *econtext;
 	TupleTableSlot *myslot;
@@ -2281,6 +2373,7 @@ CopyFrom(CopyState cstate)
 	 * only hint about them in the view case.)
 	 */
 	if (cstate->rel->rd_rel->relkind != RELKIND_RELATION &&
+		cstate->rel->rd_rel->relkind != RELKIND_PARTITIONED_TABLE &&
 		!(cstate->rel->trigdesc &&
 		  cstate->rel->trigdesc->trig_insert_instead_row))
 	{
@@ -2391,6 +2484,7 @@ CopyFrom(CopyState cstate)
 	InitResultRelInfo(resultRelInfo,
 					  cstate->rel,
 					  1,		/* dummy rangetable index */
+					  true,		/* do load partition check expression */
 					  0);
 
 	ExecOpenIndices(resultRelInfo, false);
@@ -2418,6 +2512,7 @@ CopyFrom(CopyState cstate)
 	if ((resultRelInfo->ri_TrigDesc != NULL &&
 		 (resultRelInfo->ri_TrigDesc->trig_insert_before_row ||
 		  resultRelInfo->ri_TrigDesc->trig_insert_instead_row)) ||
+		cstate->partition_tree != NULL ||
 		cstate->volatile_defexprs)
 	{
 		useHeapMultiInsert = false;
@@ -2439,10 +2534,46 @@ CopyFrom(CopyState cstate)
 	 */
 	ExecBSInsertTriggers(estate, resultRelInfo);
 
+	/* Initialize FDW partition insert plans */
+	if (cstate->partition_tree)
+	{
+		int			i,
+					j;
+		List	   *fdw_private_lists = cstate->partition_fdw_priv_lists;
+		ModifyTableState   *mtstate = makeNode(ModifyTableState);
+		ResultRelInfo	   *leaf_part_rri;
+
+		/* Mostly dummy containing enough state for BeginForeignModify */
+		mtstate->ps.state = estate;
+		mtstate->operation = CMD_INSERT;
+
+		j = 0;
+		leaf_part_rri = cstate->partitions;
+		for (i = 0; i < cstate->num_partitions; i++)
+		{
+			if (leaf_part_rri->ri_FdwRoutine)
+			{
+				List *fdw_private;
+
+				Assert(fdw_private_lists);
+				fdw_private = list_nth(fdw_private_lists, j++);
+				leaf_part_rri->ri_FdwRoutine->BeginForeignModify(mtstate,
+															leaf_part_rri,
+															fdw_private,
+															0, 0);
+			}
+			leaf_part_rri++;
+		}
+	}
+
 	values = (Datum *) palloc(tupDesc->natts * sizeof(Datum));
 	nulls = (bool *) palloc(tupDesc->natts * sizeof(bool));
 
-	bistate = GetBulkInsertState();
+	if (useHeapMultiInsert)
+		bistate = GetBulkInsertState();
+	else
+		bistate = NULL;
+
 	econtext = GetPerTupleExprContext(estate);
 
 	/* Set up callback to identify error line number */
@@ -2494,6 +2625,50 @@ CopyFrom(CopyState cstate)
 		slot = myslot;
 		ExecStoreTuple(tuple, slot, InvalidBuffer, false);
 
+		/* Determine the partition to heap_insert the tuple into */
+		if (cstate->partition_tree)
+		{
+			int		leaf_part_index;
+			TupleConversionMap *map;
+
+			/*
+			 * Away we go ... If we end up not finding a partition after all,
+			 * ExecFindPartition() does not return and errors out instead.
+			 * Otherwise, the returned value is to be used as an index into
+			 * arrays mt_partitions[] and mt_partition_tupconv_maps[] that
+			 * will get us the ResultRelInfo and TupleConversionMap for the
+			 * partition, respectively.
+			 */
+			leaf_part_index = ExecFindPartition(resultRelInfo,
+												cstate->partition_tree,
+												slot,
+												estate);
+			Assert(leaf_part_index >= 0 &&
+				   leaf_part_index < cstate->num_partitions);
+
+			/*
+			 * Save the old ResultRelInfo and switch to the one corresponding
+			 * to the selected partition.
+			 */
+			saved_resultRelInfo = resultRelInfo;
+			resultRelInfo = cstate->partitions + leaf_part_index;
+
+			/*
+			 * For ExecInsertIndexTuples() to work on the partition's indexes
+			 */
+			estate->es_result_relation_info = resultRelInfo;
+
+			/*
+			 * We might need to convert from the parent rowtype to the
+			 * partition rowtype.
+			 */
+			map = cstate->partition_tupconv_maps[leaf_part_index];
+			if (map)
+				tuple = do_convert_tuple(tuple, map);
+
+			tuple->t_tableOid = RelationGetRelid(resultRelInfo->ri_RelationDesc);
+		}
+
 		skip_tuple = false;
 
 		/* BEFORE ROW INSERT Triggers */
@@ -2523,7 +2698,16 @@ CopyFrom(CopyState cstate)
 					resultRelInfo->ri_PartitionCheck)
 					ExecConstraints(resultRelInfo, slot, estate);
 
-				if (useHeapMultiInsert)
+				if (resultRelInfo->ri_FdwRoutine)
+				{
+					resultRelInfo->ri_FdwRoutine->ExecForeignInsert(estate,
+																resultRelInfo,
+																	slot,
+																	NULL);
+					/* AFTER ROW INSERT Triggers */
+					ExecARInsertTriggers(estate, resultRelInfo, tuple, NIL);
+				}
+				else if (useHeapMultiInsert)
 				{
 					/* Add this tuple to the tuple buffer */
 					if (nBufferedTuples == 0)
@@ -2553,7 +2737,8 @@ CopyFrom(CopyState cstate)
 					List	   *recheckIndexes = NIL;
 
 					/* OK, store the tuple and create index entries for it */
-					heap_insert(cstate->rel, tuple, mycid, hi_options, bistate);
+					heap_insert(resultRelInfo->ri_RelationDesc, tuple, mycid,
+								hi_options, bistate);
 
 					if (resultRelInfo->ri_NumIndices > 0)
 						recheckIndexes = ExecInsertIndexTuples(slot,
@@ -2577,6 +2762,12 @@ CopyFrom(CopyState cstate)
 			 * tuples inserted by an INSERT command.
 			 */
 			processed++;
+
+			if (saved_resultRelInfo)
+			{
+				resultRelInfo = saved_resultRelInfo;
+				estate->es_result_relation_info = resultRelInfo;
+			}
 		}
 	}
 
@@ -2590,7 +2781,8 @@ CopyFrom(CopyState cstate)
 	/* Done, clean up */
 	error_context_stack = errcallback.previous;
 
-	FreeBulkInsertState(bistate);
+	if (bistate)
+		FreeBulkInsertState(bistate);
 
 	MemoryContextSwitchTo(oldcontext);
 
@@ -2614,6 +2806,28 @@ CopyFrom(CopyState cstate)
 
 	ExecCloseIndices(resultRelInfo);
 
+	/*
+	 * Close all partitions and indices thereof, also clean up any
+	 * state of FDW modify plans.
+	 */
+	if (cstate->partition_tree)
+	{
+		int		i;
+
+		for (i = 0; i < cstate->num_partitions; i++)
+		{
+			ResultRelInfo *resultRelInfo = cstate->partitions + i;
+
+			ExecCloseIndices(resultRelInfo);
+			heap_close(resultRelInfo->ri_RelationDesc, NoLock);
+
+			if (resultRelInfo->ri_FdwRoutine &&
+				resultRelInfo->ri_FdwRoutine->EndForeignModify != NULL)
+				resultRelInfo->ri_FdwRoutine->EndForeignModify(estate,
+														   resultRelInfo);
+		}
+	}
+
 	FreeExecutorState(estate);
 
 	/*
diff --git a/src/backend/commands/tablecmds.c b/src/backend/commands/tablecmds.c
index 3b72ae3..6478211 100644
--- a/src/backend/commands/tablecmds.c
+++ b/src/backend/commands/tablecmds.c
@@ -1293,6 +1293,7 @@ ExecuteTruncate(TruncateStmt *stmt)
 		InitResultRelInfo(resultRelInfo,
 						  rel,
 						  0,	/* dummy rangetable index */
+						  false,
 						  0);
 		resultRelInfo++;
 	}
diff --git a/src/backend/executor/execMain.c b/src/backend/executor/execMain.c
index ea3f59a..3b4dd38 100644
--- a/src/backend/executor/execMain.c
+++ b/src/backend/executor/execMain.c
@@ -826,6 +826,7 @@ InitPlan(QueryDesc *queryDesc, int eflags)
 			InitResultRelInfo(resultRelInfo,
 							  resultRelation,
 							  resultRelationIndex,
+							  true,
 							  estate->es_instrument);
 			resultRelInfo++;
 		}
@@ -1215,6 +1216,7 @@ void
 InitResultRelInfo(ResultRelInfo *resultRelInfo,
 				  Relation resultRelationDesc,
 				  Index resultRelationIndex,
+				  bool load_partition_check,
 				  int instrument_options)
 {
 	MemSet(resultRelInfo, 0, sizeof(ResultRelInfo));
@@ -1252,8 +1254,10 @@ InitResultRelInfo(ResultRelInfo *resultRelInfo,
 	resultRelInfo->ri_ConstraintExprs = NULL;
 	resultRelInfo->ri_junkFilter = NULL;
 	resultRelInfo->ri_projectReturning = NULL;
-	resultRelInfo->ri_PartitionCheck =
-						RelationGetPartitionQual(resultRelationDesc, true);
+	if (load_partition_check)
+		resultRelInfo->ri_PartitionCheck =
+							RelationGetPartitionQual(resultRelationDesc,
+													 true);
 }
 
 /*
@@ -1316,6 +1320,7 @@ ExecGetTriggerResultRel(EState *estate, Oid relid)
 	InitResultRelInfo(rInfo,
 					  rel,
 					  0,		/* dummy rangetable index */
+					  true,
 					  estate->es_instrument);
 	estate->es_trig_target_relations =
 		lappend(estate->es_trig_target_relations, rInfo);
@@ -2996,3 +3001,53 @@ EvalPlanQualEnd(EPQState *epqstate)
 	epqstate->planstate = NULL;
 	epqstate->origslot = NULL;
 }
+
+/*
+ * ExecFindPartition -- Find a leaf partition in the partition tree rooted
+ * at parent, for the heap tuple contained in *slot
+ *
+ * estate must be non-NULL; we'll need it to compute any expressions in the
+ * partition key(s)
+ *
+ * If no leaf partition is found, this routine errors out with the appropriate
+ * error message, else it returns the leaf partition sequence number returned
+ * by get_partition_for_tuple() unchanged.
+ */
+int
+ExecFindPartition(ResultRelInfo *resultRelInfo, PartitionTreeNode parent,
+				  TupleTableSlot *slot, EState *estate)
+{
+	int		result;
+	Oid		failed_at;
+	ExprContext *econtext = GetPerTupleExprContext(estate);
+
+	econtext->ecxt_scantuple = slot;
+	result = get_partition_for_tuple(parent, slot, estate, &failed_at);
+
+	if (result < 0)
+	{
+		Relation	rel = resultRelInfo->ri_RelationDesc;
+		char	   *val_desc;
+		Bitmapset  *insertedCols,
+				   *updatedCols,
+				   *modifiedCols;
+		TupleDesc	tupDesc = RelationGetDescr(rel);
+
+		insertedCols = GetInsertedColumns(resultRelInfo, estate);
+		updatedCols = GetUpdatedColumns(resultRelInfo, estate);
+		modifiedCols = bms_union(insertedCols, updatedCols);
+		val_desc = ExecBuildSlotValueDescription(RelationGetRelid(rel),
+												 slot,
+												 tupDesc,
+												 modifiedCols,
+												 64);
+		Assert(OidIsValid(failed_at));
+		ereport(ERROR,
+				(errcode(ERRCODE_CHECK_VIOLATION),
+				 errmsg("no partition of relation \"%s\" found for row",
+						get_rel_name(failed_at)),
+		  val_desc ? errdetail("Failing row contains %s.", val_desc) : 0));
+	}
+
+	return result;
+}
diff --git a/src/backend/executor/nodeModifyTable.c b/src/backend/executor/nodeModifyTable.c
index a612b08..3523a2d 100644
--- a/src/backend/executor/nodeModifyTable.c
+++ b/src/backend/executor/nodeModifyTable.c
@@ -258,6 +258,7 @@ ExecInsert(ModifyTableState *mtstate,
 {
 	HeapTuple	tuple;
 	ResultRelInfo *resultRelInfo;
+	ResultRelInfo *saved_resultRelInfo = NULL;
 	Relation	resultRelationDesc;
 	Oid			newId;
 	List	   *recheckIndexes = NIL;
@@ -272,6 +273,50 @@ ExecInsert(ModifyTableState *mtstate,
 	 * get information on the (current) result relation
 	 */
 	resultRelInfo = estate->es_result_relation_info;
+
+	/* Determine the partition to heap_insert the tuple into */
+	if (mtstate->mt_partition_tree)
+	{
+		int		leaf_part_index;
+		TupleConversionMap *map;
+
+		/*
+		 * Away we go ... If we end up not finding a partition after all,
+		 * ExecFindPartition() does not return and errors out instead.
+		 * Otherwise, the returned value is to be used as an index into
+		 * arrays mt_partitions[] and mt_partition_tupconv_maps[] that
+		 * will get us the ResultRelInfo and TupleConversionMap for the
+		 * partition, respectively.
+		 */
+		leaf_part_index = ExecFindPartition(resultRelInfo,
+											mtstate->mt_partition_tree,
+											slot,
+											estate);
+		Assert(leaf_part_index >= 0 &&
+			   leaf_part_index < mtstate->mt_num_partitions);
+
+		/*
+		 * Save the old ResultRelInfo and switch to the one corresponding to
+		 * the selected partition.
+		 */
+		saved_resultRelInfo = resultRelInfo;
+		resultRelInfo = mtstate->mt_partitions + leaf_part_index;
+
+		/* For ExecInsertIndexTuples() to work on the partition's indexes */
+		estate->es_result_relation_info = resultRelInfo;
+
+		/*
+		 * We might need to convert from the parent rowtype to the partition
+		 * rowtype.
+		 */
+		map = mtstate->mt_partition_tupconv_maps[leaf_part_index];
+		if (map)
+		{
+			tuple = do_convert_tuple(tuple, map);
+			ExecStoreTuple(tuple, slot, InvalidBuffer, false);
+		}
+	}
+
 	resultRelationDesc = resultRelInfo->ri_RelationDesc;
 
 	/*
@@ -511,6 +556,12 @@ ExecInsert(ModifyTableState *mtstate,
 
 	list_free(recheckIndexes);
 
+	if (saved_resultRelInfo)
+	{
+		resultRelInfo = saved_resultRelInfo;
+		estate->es_result_relation_info = resultRelInfo;
+	}
+
 	/*
 	 * Check any WITH CHECK OPTION constraints from parent views.  We are
 	 * required to do this after testing all constraints and uniqueness
@@ -1565,6 +1616,7 @@ ExecInitModifyTable(ModifyTable *node, EState *estate, int eflags)
 	Plan	   *subplan;
 	ListCell   *l;
 	int			i;
+	Relation	rel;
 
 	/* check for unsupported flags */
 	Assert(!(eflags & (EXEC_FLAG_BACKWARD | EXEC_FLAG_MARK)));
@@ -1655,6 +1707,100 @@ ExecInitModifyTable(ModifyTable *node, EState *estate, int eflags)
 
 	estate->es_result_relation_info = saved_resultRelInfo;
 
+	/* Build state for INSERT tuple routing */
+	rel = mtstate->resultRelInfo->ri_RelationDesc;
+	if (operation == CMD_INSERT &&
+		rel->rd_rel->relkind == RELKIND_PARTITIONED_TABLE)
+	{
+		int					i,
+							j,
+							num_leaf_parts;
+		List			   *leaf_parts;
+		ListCell		   *cell;
+		ResultRelInfo	   *leaf_part_rri;
+
+		/* Form the partition node tree and lock partitions */
+		mtstate->mt_partition_tree = RelationGetPartitionTreeNode(rel,
+														RowExclusiveLock);
+		leaf_parts = get_leaf_partition_oids(mtstate->mt_partition_tree);
+		num_leaf_parts = list_length(leaf_parts);
+
+		mtstate->mt_num_partitions = num_leaf_parts;
+		mtstate->mt_partitions = (ResultRelInfo *)
+						palloc0(num_leaf_parts * sizeof(ResultRelInfo));
+		mtstate->mt_partition_tupconv_maps = (TupleConversionMap **)
+					palloc0(num_leaf_parts * sizeof(TupleConversionMap *));
+
+		leaf_part_rri = mtstate->mt_partitions;
+		i = j = 0;
+		foreach(cell, leaf_parts)
+		{
+			Oid			ftoid = lfirst_oid(cell);
+			Relation	part_rel;
+
+			part_rel = heap_open(ftoid, RowExclusiveLock);
+
+			/*
+			 * Verify result relation is a valid target for the current
+			 * operation
+			 */
+			CheckValidResultRel(part_rel, CMD_INSERT);
+
+			InitResultRelInfo(leaf_part_rri,
+							  part_rel,
+							  1,		/* dummy */
+							  false,	/* no need for partition checks */
+							  eflags);
+
+			/* Open partition indices (note: ON CONFLICT unsupported)*/
+			if (leaf_part_rri->ri_RelationDesc->rd_rel->relhasindex &&
+				operation != CMD_DELETE &&
+				leaf_part_rri->ri_IndexRelationDescs == NULL)
+				ExecOpenIndices(leaf_part_rri, false);
+
+			if (leaf_part_rri->ri_FdwRoutine)
+			{
+				ListCell    *lc;
+				List	    *fdw_private;
+				int			 k;
+
+				/*
+				 * There are as many fdw_private's in fdwPrivLists as there
+				 * are FDW partitions, but we must find the intended for the
+				 * this foreign table.
+				 */
+				k = 0;
+				foreach(lc, node->fdwPartitionOids)
+				{
+					if (lfirst_oid(lc) == ftoid)
+						break;
+					k++;
+				}
+
+				Assert(k < num_leaf_parts);
+				fdw_private = (List *) list_nth(node->fdwPrivLists, k);
+				Assert(fdw_private != NIL);
+
+				leaf_part_rri->ri_FdwRoutine->BeginForeignModify(mtstate,
+																leaf_part_rri,
+																fdw_private,
+																0,
+																eflags);
+				j++;
+			}
+
+			if (!equalTupleDescs(RelationGetDescr(rel),
+								 RelationGetDescr(part_rel)))
+				mtstate->mt_partition_tupconv_maps[i] =
+							convert_tuples_by_name(RelationGetDescr(rel),
+												   RelationGetDescr(part_rel),
+								  gettext_noop("could not convert row type"));
+
+			leaf_part_rri++;
+			i++;
+		}
+	}
+
 	/*
 	 * Initialize any WITH CHECK OPTION constraints if needed.
 	 */
@@ -1972,6 +2118,20 @@ ExecEndModifyTable(ModifyTableState *node)
 														   resultRelInfo);
 	}
 
+	/* Close all partitions and indices thereof */
+	for (i = 0; i < node->mt_num_partitions; i++)
+	{
+		ResultRelInfo *resultRelInfo = node->mt_partitions + i;
+
+		ExecCloseIndices(resultRelInfo);
+		heap_close(resultRelInfo->ri_RelationDesc, NoLock);
+
+		if (resultRelInfo->ri_FdwRoutine &&
+			resultRelInfo->ri_FdwRoutine->EndForeignModify != NULL)
+			resultRelInfo->ri_FdwRoutine->EndForeignModify(node->ps.state,
+														   resultRelInfo);
+	}
+
 	/*
 	 * Free the exprcontext
 	 */
diff --git a/src/backend/nodes/copyfuncs.c b/src/backend/nodes/copyfuncs.c
index 28d0036..470dee7 100644
--- a/src/backend/nodes/copyfuncs.c
+++ b/src/backend/nodes/copyfuncs.c
@@ -188,6 +188,7 @@ _copyModifyTable(const ModifyTable *from)
 	COPY_NODE_FIELD(withCheckOptionLists);
 	COPY_NODE_FIELD(returningLists);
 	COPY_NODE_FIELD(fdwPrivLists);
+	COPY_NODE_FIELD(fdwPartitionOids);
 	COPY_BITMAPSET_FIELD(fdwDirectModifyPlans);
 	COPY_NODE_FIELD(rowMarks);
 	COPY_SCALAR_FIELD(epqParam);
diff --git a/src/backend/nodes/outfuncs.c b/src/backend/nodes/outfuncs.c
index 0d858f5..5c8eced 100644
--- a/src/backend/nodes/outfuncs.c
+++ b/src/backend/nodes/outfuncs.c
@@ -340,6 +340,7 @@ _outModifyTable(StringInfo str, const ModifyTable *node)
 	WRITE_NODE_FIELD(withCheckOptionLists);
 	WRITE_NODE_FIELD(returningLists);
 	WRITE_NODE_FIELD(fdwPrivLists);
+	WRITE_NODE_FIELD(fdwPartitionOids);
 	WRITE_BITMAPSET_FIELD(fdwDirectModifyPlans);
 	WRITE_NODE_FIELD(rowMarks);
 	WRITE_INT_FIELD(epqParam);
diff --git a/src/backend/nodes/readfuncs.c b/src/backend/nodes/readfuncs.c
index c587d4e..ddd78c7 100644
--- a/src/backend/nodes/readfuncs.c
+++ b/src/backend/nodes/readfuncs.c
@@ -1495,6 +1495,7 @@ _readModifyTable(void)
 	READ_NODE_FIELD(withCheckOptionLists);
 	READ_NODE_FIELD(returningLists);
 	READ_NODE_FIELD(fdwPrivLists);
+	READ_NODE_FIELD(fdwPartitionOids);
 	READ_BITMAPSET_FIELD(fdwDirectModifyPlans);
 	READ_NODE_FIELD(rowMarks);
 	READ_INT_FIELD(epqParam);
diff --git a/src/backend/optimizer/plan/createplan.c b/src/backend/optimizer/plan/createplan.c
index ad49674..caf2f44 100644
--- a/src/backend/optimizer/plan/createplan.c
+++ b/src/backend/optimizer/plan/createplan.c
@@ -22,6 +22,7 @@
 #include "access/stratnum.h"
 #include "access/sysattr.h"
 #include "catalog/pg_class.h"
+#include "catalog/pg_inherits_fn.h"
 #include "foreign/fdwapi.h"
 #include "miscadmin.h"
 #include "nodes/extensible.h"
@@ -6159,6 +6160,81 @@ make_modifytable(PlannerInfo *root,
 	node->fdwPrivLists = fdw_private_list;
 	node->fdwDirectModifyPlans = direct_modify_plans;
 
+	/* Collect insert plans for all FDW-managed partitions */
+	if (node->operation == CMD_INSERT)
+	{
+		RangeTblEntry  *rte,
+					  **saved_simple_rte_array;
+		List		   *partition_oids,
+					   *fdw_partition_oids;
+
+		Assert(list_length(resultRelations) == 1);
+		rte = rt_fetch(linitial_int(resultRelations), root->parse->rtable);
+		Assert(rte->rtekind == RTE_RELATION);
+
+		if (rte->relkind != RELKIND_PARTITIONED_TABLE)
+			return node;
+
+		partition_oids = find_all_inheritors(rte->relid, RowExclusiveLock,
+											 NULL);
+
+		/* Discard any previous content which is useless anyway */
+		fdw_private_list = NIL;
+		fdw_partition_oids = NIL;
+
+		/*
+		 * To force the FDW driver fetch the intended RTE, we need to temporarily
+		 * switch root->simple_rte_array to the one holding only that RTE at a
+		 * designated index, for every foreign table.
+		 */
+		saved_simple_rte_array = root->simple_rte_array;
+		root->simple_rte_array = (RangeTblEntry **)
+										palloc0(2 * sizeof(RangeTblEntry *));
+		foreach(lc, partition_oids)
+		{
+			Oid		myoid = lfirst_oid(lc);
+			FdwRoutine *fdwroutine;
+			List	   *fdw_private;
+
+			/*
+			 * We are only interested in foreign tables.  Note that this will
+			 * also eliminate any partitioned tables since foreign tables can
+			 * only ever be leaf partitions.
+			 */
+			if (get_rel_relkind(myoid) != RELKIND_FOREIGN_TABLE)
+				continue;
+
+			fdw_partition_oids = lappend_oid(fdw_partition_oids, myoid);
+
+			fdwroutine = GetFdwRoutineByRelId(myoid);
+			if (fdwroutine && fdwroutine->PlanForeignModify)
+			{
+				RangeTblEntry *fdw_rte;
+
+				fdw_rte = copyObject(rte);
+				fdw_rte->relid = myoid;
+				fdw_rte->relkind = RELKIND_FOREIGN_TABLE;
+
+				/*
+				 * Assumes PlanForeignModify() uses planner_rt_fetch(), also,
+				 * see the above comment.
+				 */
+				root->simple_rte_array[1] = fdw_rte;
+
+				fdw_private = fdwroutine->PlanForeignModify(root, node, 1, 0);
+			}
+			else
+				fdw_private = NIL;
+
+			fdw_private_list = lappend(fdw_private_list, fdw_private);
+		}
+
+		root->simple_rte_array = saved_simple_rte_array;
+
+		node->fdwPrivLists = fdw_private_list;
+		node->fdwPartitionOids = fdw_partition_oids;
+	}
+
 	return node;
 }
 
diff --git a/src/backend/parser/analyze.c b/src/backend/parser/analyze.c
index 6901e08..c10b6c3 100644
--- a/src/backend/parser/analyze.c
+++ b/src/backend/parser/analyze.c
@@ -798,8 +798,16 @@ transformInsertStmt(ParseState *pstate, InsertStmt *stmt)
 
 	/* Process ON CONFLICT, if any. */
 	if (stmt->onConflictClause)
+	{
+		/* Bail out if target relation is partitioned table */
+		if (pstate->p_target_rangetblentry->relkind == RELKIND_PARTITIONED_TABLE)
+			ereport(ERROR,
+					(errcode(ERRCODE_FEATURE_NOT_SUPPORTED),
+					 errmsg("ON CONFLICT clause is not supported with partitioned tables")));
+
 		qry->onConflict = transformOnConflictClause(pstate,
 													stmt->onConflictClause);
+	}
 
 	/*
 	 * If we have a RETURNING clause, we need to add the target relation to
diff --git a/src/include/catalog/partition.h b/src/include/catalog/partition.h
index 672183b..e1ccaf8 100644
--- a/src/include/catalog/partition.h
+++ b/src/include/catalog/partition.h
@@ -14,6 +14,8 @@
 #define PARTITION_H
 
 #include "fmgr.h"
+#include "executor/tuptable.h"
+#include "nodes/execnodes.h"
 #include "parser/parse_node.h"
 #include "utils/rel.h"
 
@@ -50,4 +52,9 @@ extern List *RelationGetPartitionQual(Relation rel, bool recurse);
 /* For tuple routing */
 extern PartitionTreeNode RelationGetPartitionTreeNode(Relation rel, int lockmode);
 extern List *get_leaf_partition_oids(PartitionTreeNode parent);
+
+extern int get_partition_for_tuple(PartitionTreeNode parent,
+					TupleTableSlot *slot,
+					EState *estate,
+					Oid *failed_at);
 #endif   /* PARTITION_H */
diff --git a/src/include/executor/executor.h b/src/include/executor/executor.h
index 136276b..fadc402 100644
--- a/src/include/executor/executor.h
+++ b/src/include/executor/executor.h
@@ -14,6 +14,7 @@
 #ifndef EXECUTOR_H
 #define EXECUTOR_H
 
+#include "catalog/partition.h"
 #include "executor/execdesc.h"
 #include "nodes/parsenodes.h"
 
@@ -188,6 +189,7 @@ extern void CheckValidResultRel(Relation resultRel, CmdType operation);
 extern void InitResultRelInfo(ResultRelInfo *resultRelInfo,
 				  Relation resultRelationDesc,
 				  Index resultRelationIndex,
+				  bool load_partition_check,
 				  int instrument_options);
 extern ResultRelInfo *ExecGetTriggerResultRel(EState *estate, Oid relid);
 extern bool ExecContextForcesOids(PlanState *planstate, bool *hasoids);
@@ -211,6 +213,10 @@ extern void EvalPlanQualSetPlan(EPQState *epqstate,
 extern void EvalPlanQualSetTuple(EPQState *epqstate, Index rti,
 					 HeapTuple tuple);
 extern HeapTuple EvalPlanQualGetTuple(EPQState *epqstate, Index rti);
+extern int ExecFindPartition(ResultRelInfo *resultRelInfo,
+				  PartitionTreeNode parent,
+				  TupleTableSlot *slot,
+				  EState *estate);
 
 #define EvalPlanQualSetSlot(epqstate, slot)  ((epqstate)->origslot = (slot))
 extern void EvalPlanQualFetchRowMarks(EPQState *epqstate);
diff --git a/src/include/nodes/execnodes.h b/src/include/nodes/execnodes.h
index ff8b66b..06675bc 100644
--- a/src/include/nodes/execnodes.h
+++ b/src/include/nodes/execnodes.h
@@ -16,6 +16,7 @@
 
 #include "access/genam.h"
 #include "access/heapam.h"
+#include "access/tupconvert.h"
 #include "executor/instrument.h"
 #include "lib/pairingheap.h"
 #include "nodes/params.h"
@@ -1147,6 +1148,15 @@ typedef struct ModifyTableState
 										 * tlist  */
 	TupleTableSlot *mt_conflproj;		/* CONFLICT ... SET ... projection
 										 * target */
+	struct PartitionTreeNodeData *mt_partition_tree;
+										/* Partition descriptor node tree */
+	ResultRelInfo  *mt_partitions;		/* Per leaf partition target
+										 * relations */
+	TupleConversionMap **mt_partition_tupconv_maps;
+										/* Per leaf partition
+										 * tuple conversion map */
+	int				mt_num_partitions;	/* Number of leaf partition target
+										 * relations in the above array */
 } ModifyTableState;
 
 /* ----------------
diff --git a/src/include/nodes/plannodes.h b/src/include/nodes/plannodes.h
index e2fbc7d..d82222c 100644
--- a/src/include/nodes/plannodes.h
+++ b/src/include/nodes/plannodes.h
@@ -189,6 +189,7 @@ typedef struct ModifyTable
 	List	   *returningLists; /* per-target-table RETURNING tlists */
 	List	   *fdwPrivLists;	/* per-target-table FDW private data lists */
 	Bitmapset  *fdwDirectModifyPlans;	/* indices of FDW DM plans */
+	List	   *fdwPartitionOids;	/* OIDs of FDW-managed partition */
 	List	   *rowMarks;		/* PlanRowMarks (non-locking only) */
 	int			epqParam;		/* ID of Param for EvalPlanQual re-eval */
 	OnConflictAction onConflictAction;	/* ON CONFLICT action */
diff --git a/src/test/regress/expected/insert.out b/src/test/regress/expected/insert.out
index 9ae6b09..d5dcb59 100644
--- a/src/test/regress/expected/insert.out
+++ b/src/test/regress/expected/insert.out
@@ -227,6 +227,58 @@ DETAIL:  Failing row contains (cc, 1).
 -- ok
 insert into part_EE_FF_1_10 values ('ff', 1);
 insert into part_EE_FF_10_20 values ('ff', 11);
+-- Check tuple routing for partitioned tables
+-- fail
+insert into range_parted values ('a', 0);
+ERROR:  no partition of relation "range_parted" found for row
+DETAIL:  Failing row contains (a, 0).
+-- ok
+insert into range_parted values ('a', 1);
+insert into range_parted values ('a', 10);
+-- fail
+insert into range_parted values ('a', 20);
+ERROR:  no partition of relation "range_parted" found for row
+DETAIL:  Failing row contains (a, 20).
+-- ok
+insert into range_parted values ('b', 1);
+insert into range_parted values ('b', 10);
+select tableoid::regclass, * from range_parted;
+    tableoid    | a | b  
+----------------+---+----
+ part_a_1_a_10  | a |  1
+ part_a_1_a_10  | a |  1
+ part_a_10_a_20 | a | 10
+ part_b_1_b_10  | b |  1
+ part_b_10_b_20 | b | 10
+ part_b_10_b_20 | b | 10
+(6 rows)
+
+-- ok
+insert into list_parted values (null, 1);
+insert into list_parted (a) values ('aA');
+-- fail (partition of part_EE_FF not found)
+insert into list_parted values ('EE', 0);
+ERROR:  no partition of relation "part_ee_ff" found for row
+DETAIL:  Failing row contains (EE, 0).
+insert into part_EE_FF values ('EE', 0);
+ERROR:  no partition of relation "part_ee_ff" found for row
+DETAIL:  Failing row contains (EE, 0).
+-- ok
+insert into list_parted values ('EE', 1);
+insert into part_EE_FF values ('EE', 10);
+select tableoid::regclass, * from list_parted;
+     tableoid     | a  | b  
+------------------+----+----
+ part_aa_bb       | aA |   
+ part_cc_dd       | cC |  1
+ part_null        |    |  0
+ part_null        |    |  1
+ part_ee_ff_1_10  | ff |  1
+ part_ee_ff_1_10  | EE |  1
+ part_ee_ff_10_20 | ff | 11
+ part_ee_ff_10_20 | EE | 10
+(8 rows)
+
 -- cleanup
 drop table range_parted cascade;
 NOTICE:  drop cascades to 4 other objects
diff --git a/src/test/regress/sql/insert.sql b/src/test/regress/sql/insert.sql
index b6e821e..fbd30d9 100644
--- a/src/test/regress/sql/insert.sql
+++ b/src/test/regress/sql/insert.sql
@@ -140,6 +140,31 @@ insert into part_EE_FF_1_10 values ('cc', 1);
 insert into part_EE_FF_1_10 values ('ff', 1);
 insert into part_EE_FF_10_20 values ('ff', 11);
 
+-- Check tuple routing for partitioned tables
+
+-- fail
+insert into range_parted values ('a', 0);
+-- ok
+insert into range_parted values ('a', 1);
+insert into range_parted values ('a', 10);
+-- fail
+insert into range_parted values ('a', 20);
+-- ok
+insert into range_parted values ('b', 1);
+insert into range_parted values ('b', 10);
+select tableoid::regclass, * from range_parted;
+
+-- ok
+insert into list_parted values (null, 1);
+insert into list_parted (a) values ('aA');
+-- fail (partition of part_EE_FF not found)
+insert into list_parted values ('EE', 0);
+insert into part_EE_FF values ('EE', 0);
+-- ok
+insert into list_parted values ('EE', 1);
+insert into part_EE_FF values ('EE', 10);
+select tableoid::regclass, * from list_parted;
+
 -- cleanup
 drop table range_parted cascade;
 drop table list_parted cascade;
-- 
1.7.1

