Logo AND Algorithmique Numérique Distribuée

Public GIT Repository
re-start smpi from scratch to use SIMIX.
authormarkls <markls@48e7efb5-ca39-0410-a469-dd3cf9ba447f>
Tue, 3 Jul 2007 05:35:31 +0000 (05:35 +0000)
committermarkls <markls@48e7efb5-ca39-0410-a469-dd3cf9ba447f>
Tue, 3 Jul 2007 05:35:31 +0000 (05:35 +0000)
git-svn-id: svn+ssh://scm.gforge.inria.fr/svn/simgrid/simgrid/trunk@3646 48e7efb5-ca39-0410-a469-dd3cf9ba447f

src/include/surf/surf.h
src/smpi/include/smpi.h
src/smpi/sample/first.c
src/smpi/sample/second.c [new file with mode: 0644]
src/smpi/scripts/smpicc
src/smpi/src/smpi_base.c
src/smpi/src/smpi_mpi.c
src/smpi/src/smpi_new.c [new file with mode: 0644]

index a35a992..f2affd4 100644 (file)
@@ -12,7 +12,7 @@
 #include "xbt/dynar.h"
 #include "xbt/dict.h"
 #include "xbt/misc.h"
-#include "gras_config.h"
+//#include "gras_config.h"
 
 SG_BEGIN_DECL()
 
index 9f67528..3a7426d 100644 (file)
@@ -1,4 +1,6 @@
-#define DEFAULT_POWER 100
+#define SMPI_DEFAULT_SPEED 100
+
+#define SMPI_RAND_SEED 5
 
 #define MPI_ANY_SOURCE -1
 
 #define MPI_ERR_TAG     8
 
 #include <stdlib.h>
-#include <msg/msg.h>
 
-typedef enum { MPI_PORT = 0, SEND_SYNC_PORT, RECV_SYNC_PORT, MAX_CHANNEL } channel_t;
+#include <simix/simix.h>
 
 // MPI_Comm
 struct smpi_mpi_communicator_t {
-  int id;
   int size;
   int barrier;
   smx_host_t *hosts;
@@ -42,7 +42,6 @@ extern smpi_mpi_status_t smpi_mpi_status_ignore;
 
 // MPI_Datatype
 struct smpi_mpi_datatype_t {
-//  int type;
   size_t size;
 };
 typedef struct smpi_mpi_datatype_t smpi_mpi_datatype_t;
@@ -55,13 +54,6 @@ extern smpi_mpi_datatype_t smpi_mpi_int;
 extern smpi_mpi_datatype_t smpi_mpi_double;
 #define MPI_DOUBLE (&smpi_mpi_double)
 
-struct smpi_waitlist_node_t {
-  smx_process_t process;
-  struct smpi_waitlist_node_t *next;
-};
-typedef struct smpi_waitlist_node_t smpi_waitlist_node_t;
-
-// FIXME: maybe it isn't appropriate to have the next pointer inside
 // MPI_Request
 struct smpi_mpi_request_t {
   void *buf;
@@ -72,9 +64,7 @@ struct smpi_mpi_request_t {
   int tag;
   smpi_mpi_communicator_t *comm;
   short int completed;
-  smpi_waitlist_node_t *waitlist;
-  struct smpi_mpi_request_t *next;
-  int fwdthrough;
+  xbt_fifo_t waitlist;
 };
 typedef struct smpi_mpi_request_t smpi_mpi_request_t;
 typedef smpi_mpi_request_t *MPI_Request;
@@ -90,32 +80,18 @@ extern smpi_mpi_op_t smpi_mpi_land;
 extern smpi_mpi_op_t smpi_mpi_sum;
 #define MPI_SUM (&smpi_mpi_sum)
 
-// smpi_received_t
-struct smpi_received_t {
-  int commid;
-  int src;
-  int dst;
-  int tag;
-  int fwdthrough;
-  void *data;
-  struct smpi_received_t *next;
+// smpi_received_message_t
+struct smpi_received_message_t {
+       smpi_mpi_communicator_t *comm;
+       int src;
+       int dst;
+       int tag;
+       void *data;
+       int fwdthrough;
 };
-typedef struct smpi_received_t smpi_received_t;
-
-// sender/receiver (called by main routine)
-int smpi_sender(int argc, char *argv[]);
-int smpi_receiver(int argc, char *argv[]);
+typedef struct smpi_received_message_t smpi_received_message_t;
 
 // smpi functions
-int smpi_comm_rank(smpi_mpi_communicator_t *comm, smx_host_t host);
-void smpi_isend(smpi_mpi_request_t*);
-void smpi_irecv(smpi_mpi_request_t*);
-void smpi_barrier(smpi_mpi_communicator_t *comm);
-void smpi_wait(smpi_mpi_request_t *request, smpi_mpi_status_t *status);
-void smpi_wait_all(int count, smpi_mpi_request_t **requests, smpi_mpi_status_t *statuses);
-void smpi_wait_all_nostatus(int count, smpi_mpi_request_t **requests);
-void smpi_bench_begin();
-void smpi_bench_end();
-int smpi_create_request(void *buf, int count, smpi_mpi_datatype_t *datatype, int src, int dst, int tag, smpi_mpi_communicator_t *comm, smpi_mpi_request_t **request);
+extern int smpi_simulated_main(int argc, char **argv);
 unsigned int smpi_sleep(unsigned int);
 void smpi_exit(int);
index d78aeac..e37fe80 100644 (file)
@@ -1,56 +1,30 @@
-/* A first simple SPMD example program using MPI                  */
+#include <stdio.h>
+#include <mpi.h>
 
-/* The program consists of on receiver process and N-1 sender     */
-/* processes. The sender processes send a message consisting      */
-/* of their process identifier (id) and the total number of       */
-/* processes (ntasks) to the receiver. The receiver process       */
-/* prints out the values it receives in the messeges from the     */
-/* senders.                                                       */
+int main(int argc, char *argv[])
+{
+       int rank, size, err;
 
-/* Compile the program with 'mpicc first.c -o first'              */
-/* To run the program, using four of the computers specified in   */
-/* your hostfile, do 'mpirun -machinefile hostfile -np 4 first    */
+       err = MPI_Init(&argc, &argv); /* Initialize MPI */
+       if (err != MPI_SUCCESS) {
+               printf("MPI initialization failed!\n");
+               exit(1);
+       }
 
-#include <stdio.h>
-#include <mpi.h>
-main(int argc, char *argv[]) {
-  const int tag = 42;         /* Message tag */
-  int id, ntasks, source_id, dest_id, err, i;
-  MPI_Status status;
-  int msg[2];     /* Message array */
-  
-  err = MPI_Init(&argc, &argv); /* Initialize MPI */
-  if (err != MPI_SUCCESS) {
-    printf("MPI initialization failed!\n");
-    exit(1);
-  }
-  err = MPI_Comm_size(MPI_COMM_WORLD, &ntasks); /* Get nr of tasks */
-  err = MPI_Comm_rank(MPI_COMM_WORLD, &id); /* Get id of this process */
-  if (ntasks < 2) {
-    printf("You have to use at least 2 processors to run this program\n");
-    MPI_Finalize();      /* Quit if there is only one processor */
-    exit(0);
-  }
-  
-  if (id == 0) {    /* Process 0 (the receiver) does this */
-    for (i=1; i<ntasks; i++) {
-      err = MPI_Recv(msg, 2, MPI_INT, MPI_ANY_SOURCE, tag, MPI_COMM_WORLD, \
-         &status);          /* Receive a message */
-      source_id = status.MPI_SOURCE;  /* Get id of sender */
-      printf("Received message %d %d from process %d\n", msg[0], msg[1], \
-       source_id);
-    }
-  }
-  else {      /* Processes 1 to N-1 (the senders) do this */
-    msg[0] = id;    /* Put own identifier in the message */
-    msg[1] = ntasks;          /* and total number of processes */
-    dest_id = 0;    /* Destination address */
-    sleep(3);
-    err = MPI_Send(msg, 2, MPI_INT, dest_id, tag, MPI_COMM_WORLD);
-  }
-  
-  err = MPI_Finalize();          /* Terminate MPI */
-  if (id==0) printf("Ready\n");
-  return 0;
-}
+       err = MPI_Comm_size(MPI_COMM_WORLD, &size);
+       if (err != MPI_SUCCESS) {
+               printf("MPI Get Communicator Size Failed!\n");
+       }
+
+       err = MPI_Comm_rank(MPI_COMM_WORLD, &rank);
+       if (err != MPI_SUCCESS) {
+               printf("MPI Get Communicator Rank Failed!\n");
+       }
 
+       if (0 == rank) {
+               printf("root node believes there are %d nodes in world.\n", size);
+       }
+
+       err = MPI_Finalize();          /* Terminate MPI */
+       return 0;
+}
diff --git a/src/smpi/sample/second.c b/src/smpi/sample/second.c
new file mode 100644 (file)
index 0000000..d78aeac
--- /dev/null
@@ -0,0 +1,56 @@
+/* A first simple SPMD example program using MPI                  */
+
+/* The program consists of on receiver process and N-1 sender     */
+/* processes. The sender processes send a message consisting      */
+/* of their process identifier (id) and the total number of       */
+/* processes (ntasks) to the receiver. The receiver process       */
+/* prints out the values it receives in the messeges from the     */
+/* senders.                                                       */
+
+/* Compile the program with 'mpicc first.c -o first'              */
+/* To run the program, using four of the computers specified in   */
+/* your hostfile, do 'mpirun -machinefile hostfile -np 4 first    */
+
+#include <stdio.h>
+#include <mpi.h>
+main(int argc, char *argv[]) {
+  const int tag = 42;         /* Message tag */
+  int id, ntasks, source_id, dest_id, err, i;
+  MPI_Status status;
+  int msg[2];     /* Message array */
+  
+  err = MPI_Init(&argc, &argv); /* Initialize MPI */
+  if (err != MPI_SUCCESS) {
+    printf("MPI initialization failed!\n");
+    exit(1);
+  }
+  err = MPI_Comm_size(MPI_COMM_WORLD, &ntasks); /* Get nr of tasks */
+  err = MPI_Comm_rank(MPI_COMM_WORLD, &id); /* Get id of this process */
+  if (ntasks < 2) {
+    printf("You have to use at least 2 processors to run this program\n");
+    MPI_Finalize();      /* Quit if there is only one processor */
+    exit(0);
+  }
+  
+  if (id == 0) {    /* Process 0 (the receiver) does this */
+    for (i=1; i<ntasks; i++) {
+      err = MPI_Recv(msg, 2, MPI_INT, MPI_ANY_SOURCE, tag, MPI_COMM_WORLD, \
+         &status);          /* Receive a message */
+      source_id = status.MPI_SOURCE;  /* Get id of sender */
+      printf("Received message %d %d from process %d\n", msg[0], msg[1], \
+       source_id);
+    }
+  }
+  else {      /* Processes 1 to N-1 (the senders) do this */
+    msg[0] = id;    /* Put own identifier in the message */
+    msg[1] = ntasks;          /* and total number of processes */
+    dest_id = 0;    /* Destination address */
+    sleep(3);
+    err = MPI_Send(msg, 2, MPI_INT, dest_id, tag, MPI_COMM_WORLD);
+  }
+  
+  err = MPI_Finalize();          /* Terminate MPI */
+  if (id==0) printf("Ready\n");
+  return 0;
+}
+
index 4aefe17..62435f4 100755 (executable)
@@ -29,8 +29,8 @@ function modsource {
 #define gettimeofday(x, y) smpi_gettimeofday(x, y)
 HEADER
   # very simplistic transform, will probably want full parser for next version
-  grep -v "mpi.h" < ${SOURCE} | perl -pe 's/main/smpi_main/;' >> ${TMPSOURCE}
-  grep -q "smpi_main" ${TMPSOURCE}
+  grep -v "mpi.h" < ${SOURCE} | perl -pe 's/main/smpi_simulated_main/;' >> ${TMPSOURCE}
+  grep -q "smpi_simulated_main" ${TMPSOURCE}
   if [ $? -eq 0 ]; then
     cat >> ${TMPSOURCE} <<FOOTER
 int main(int argc, char **argv) {
@@ -51,7 +51,7 @@ while [ -n "$1" ]; do
       LINKARGS=""
       CMDLINE="${CMDLINE} -c "
   elif [ "${ARG%.c}" != "${ARG}" ]; then
-    INCLUDEARGS="-I${SMPI_INCLUDE} -I${SIMGRID_INCLUDE} "
+    INCLUDEARGS="-I ../include -I${SMPI_INCLUDE} -I${SIMGRID_INCLUDE} "
     SRCFILE="$(realpath ${ARG})"
     modsource ${SRCFILE}
     CMDLINE="${CMDLINE} ${TMPDIR}${SRCFILE} "
index 543a029..6528100 100644 (file)
@@ -1,22 +1,14 @@
 #include <stdio.h>
+
+#include <signal.h>
 #include <sys/time.h>
-#include "msg/msg.h"
-#include "simix/simix.h"
-#include "xbt/sysdep.h"
 #include "xbt/xbt_portability.h"
+#include "simix/simix.h"
 #include "smpi.h"
 
-smpi_mpi_request_t **smpi_pending_send_requests      = NULL;
-smpi_mpi_request_t **smpi_last_pending_send_requests = NULL;
-
-smpi_mpi_request_t **smpi_pending_recv_requests      = NULL;
-smpi_mpi_request_t **smpi_last_pending_recv_requests = NULL;
-
-smpi_received_t **smpi_received                      = NULL;
-smpi_received_t **smpi_last_received                 = NULL;
-
-smx_process_t *smpi_sender_processes                 = NULL;
-smx_process_t *smpi_receiver_processes               = NULL;
+xbt_fifo_t *smpi_pending_send_requests      = NULL;
+xbt_fifo_t *smpi_pending_recv_requests      = NULL;
+xbt_fifo_t *smpi_received_messages          = NULL;
 
 int smpi_running_hosts = 0;
 
@@ -33,580 +25,250 @@ smpi_mpi_op_t smpi_mpi_sum;
 
 static xbt_os_timer_t smpi_timer;
 static int smpi_benchmarking;
-static double smpi_reference;
-
-int smpi_run_simulation(int *argc, char **argv) {
-         smx_cond_t cond = NULL;
-         smx_action_t smx_action;
-         xbt_fifo_t actions_done = xbt_fifo_new();
-         xbt_fifo_t actions_failed = xbt_fifo_new();
-
-         srand(SEED);
-         SIMIX_global_init(&argc, argv);
-         SIMIX_function_register("smpi_main",     smpi_main);
-         SIMIX_function_register("smpi_sender",   smpi_sender);
-         SIMIX_function_register("smpi_receiver", smpi_receiver);
-  SIMIX_create_environment(argv[1]);
-  SIMIX_launch_application(argv[2]);
-
-         /* Prepare to display some more info when dying on Ctrl-C pressing */
-         signal(SIGINT,inthandler);
-
-         /* Clean IO before the run */
-         fflush(stdout);
-         fflush(stderr);
-
-         //surf_solve(); /* Takes traces into account. Returns 0.0 */
-         /* xbt_fifo_size(msg_global->process_to_run) */
-
-         while (SIMIX_solve(actions_done, actions_failed) != -1.0) {
-
-                 while ( (smx_action = xbt_fifo_pop(actions_failed)) ) {
-
-
-                         DEBUG1("** %s failed **",smx_action->name);
-                         while ( (cond = xbt_fifo_pop(smx_action->cond_list)) ) {
-                                 SIMIX_cond_broadcast(cond);
-                         }
-                         /* action finished, destroy it */
-                 //      SIMIX_action_destroy(smx_action);
-                 }
-
-                 while ( (smx_action = xbt_fifo_pop(actions_done)) ) {
-
-                         DEBUG1("** %s done **",smx_action->name);
-                         while ( (cond = xbt_fifo_pop(smx_action->cond_list)) ) {
-                                 SIMIX_cond_broadcast(cond);
-                         }
-                         /* action finished, destroy it */
-                         //SIMIX_action_destroy(smx_action);
-                 }
-         }
-         xbt_fifo_free(actions_failed);
-         xbt_fifo_free(actions_done);
- INFO1("simulation time %g", SIMIX_get_clock());
-  SIMIX_clean();
-  return 0;
-}
-
-void smpi_mpi_land_func(void *x, void *y, void *z) {
-  *(int *)z = *(int *)x && *(int *)y;
-}
-
-void smpi_mpi_sum_func(void *x, void *y, void *z) {
-  *(int *)z = *(int *)x + *(int *)y;
-}
+static double smpi_reference_speed;
 
-void smpi_mpi_init() {
-  int i;
-  int size, rank;
-  smx_host_t *hosts;
-  smx_host_t host;
-  double duration;
-  m_task_t mtask;
-
-  // will eventually need mutex
-  smpi_running_hosts++;
-
-  // initialize some local variables
-  size  = SIMIX_host_get_number();
-  host  = SIMIX_host_self();
-  hosts = SIMIX_host_get_table();
-  for(i = 0; i < size && host != hosts[i]; i++);
-  rank  = i;
-
-  // node 0 sets the globals
-  if (0 == rank) {
-
-    // global communicator
-    smpi_mpi_comm_world.id           = 0;
-    smpi_mpi_comm_world.size         = size;
-    smpi_mpi_comm_world.barrier      = 0;
-    smpi_mpi_comm_world.hosts        = hosts;
-    smpi_mpi_comm_world.processes    = xbt_malloc(sizeof(m_process_t) * size);
-    smpi_mpi_comm_world.processes[0] = SIMIX_process_self();
-
-    // mpi datatypes
-    smpi_mpi_byte.size               = (size_t)1;
-    smpi_mpi_int.size                = sizeof(int);
-    smpi_mpi_double.size             = sizeof(double);
-
-    // mpi operations
-    smpi_mpi_land.func               = &smpi_mpi_land_func;
-    smpi_mpi_sum.func                = &smpi_mpi_sum_func;
-
-    // smpi globals
-    smpi_pending_send_requests       = xbt_malloc(sizeof(smpi_mpi_request_t*) * size);
-    smpi_last_pending_send_requests  = xbt_malloc(sizeof(smpi_mpi_request_t*) * size);
-    smpi_pending_recv_requests       = xbt_malloc(sizeof(smpi_mpi_request_t*) * size);
-    smpi_last_pending_recv_requests  = xbt_malloc(sizeof(smpi_mpi_request_t*) * size);
-    smpi_received                    = xbt_malloc(sizeof(smpi_received_t*) * size);
-    smpi_last_received               = xbt_malloc(sizeof(smpi_received_t*) * size);
-    smpi_sender_processes            = xbt_malloc(sizeof(m_process_t) * size);
-    smpi_receiver_processes          = xbt_malloc(sizeof(m_process_t) * size);
-    for(i = 0; i < size; i++) {
-      smpi_pending_send_requests[i]      = NULL;
-      smpi_last_pending_send_requests[i] = NULL;
-      smpi_pending_recv_requests[i]      = NULL;
-      smpi_last_pending_recv_requests[i] = NULL;
-      smpi_received[i]                   = NULL;
-      smpi_last_received[i]              = NULL;
-    }
-    smpi_timer                      = xbt_os_timer_new();
-    smpi_reference                  = DEFAULT_POWER;
-    smpi_benchmarking               = 0;
-
-    // tell send/recv nodes to begin
-    for(i = 0; i < size; i++) {
-      mtask = MSG_task_create("READY", 0, 0, NULL);
-      MSG_task_put(mtask, hosts[i], SEND_SYNC_PORT);
-      mtask = (m_task_t)0;
-      MSG_task_get_from_host(&mtask, SEND_SYNC_PORT, hosts[i]);
-      MSG_task_destroy(mtask);
-      mtask = MSG_task_create("READY", 0, 0, NULL);
-      MSG_task_put(mtask, hosts[i], RECV_SYNC_PORT);
-      mtask = (m_task_t)0;
-      MSG_task_get_from_host(&mtask, RECV_SYNC_PORT, hosts[i]);
-      MSG_task_destroy(mtask);
-    }
-
-    // now everyone else
-    for(i = 1; i < size; i++) {
-      mtask = MSG_task_create("READY", 0, 0, NULL);
-      MSG_task_put(mtask, hosts[i], MPI_PORT);
-    }
-
-  } else {
-    // everyone needs to wait for node 0 to finish
-    mtask = (m_task_t)0;
-    MSG_task_get(&mtask, MPI_PORT);
-    MSG_task_destroy(mtask);
-    smpi_mpi_comm_world.processes[rank] = SIMIX_process_self();
-  }
-
-  // now that mpi_comm_world_processes is set, it's safe to set a barrier
-  smpi_barrier(&smpi_mpi_comm_world);
-}
+XBT_LOG_NEW_DEFAULT_CATEGORY(smpi, "SMPI");
 
-void smpi_mpi_finalize() {
-  int i;
-  smpi_running_hosts--;
-  if (0 <= smpi_running_hosts) {
-    for(i = 0; i < smpi_mpi_comm_world.size; i++) {
-      if(SIMIX_process_is_suspended(smpi_sender_processes[i])) {
-        SIMIX_process_resume(smpi_sender_processes[i]);
-      }
-      if(SIMIX_process_is_suspended(smpi_receiver_processes[i])) {
-        SIMIX_process_resume(smpi_receiver_processes[i]);
-      }
-    }
-  } else {
-    xbt_free(smpi_mpi_comm_world.processes);
-    xbt_free(smpi_pending_send_requests);
-    xbt_free(smpi_last_pending_send_requests);
-    xbt_free(smpi_pending_recv_requests);
-    xbt_free(smpi_last_pending_recv_requests);
-    xbt_free(smpi_received);
-    xbt_free(smpi_last_received);
-    xbt_free(smpi_sender_processes);
-    xbt_free(smpi_receiver_processes);
-    xbt_os_timer_free(smpi_timer);
-  }
+void smpi_sender()
+{
+       return;
 }
 
-void smpi_complete(smpi_mpi_request_t *request) {
-  smpi_waitlist_node_t *current, *next;
-  request->completed = 1;
-  request->next      = NULL;
-  current = request->waitlist;
-  while(NULL != current) {
-    if(SIMIX_process_is_suspended(current->process)) {
-      SIMIX_process_resume(current->process);
-    }
-    next = current->next;
-    xbt_free(current);
-    current = next;
-  }
-  request->waitlist  = NULL;
+void smpi_receiver()
+{
+       return;
 }
 
-int smpi_host_rank_self() {
-  return smpi_comm_rank(&smpi_mpi_comm_world, SIMIX_host_self());
+int smpi_run_simulation(int argc, char **argv)
+{
+       smx_cond_t   cond           = NULL;
+       smx_action_t action         = NULL;
+
+       xbt_fifo_t   actions_failed = xbt_fifo_new();
+       xbt_fifo_t   actions_done   = xbt_fifo_new();
+
+       srand(SMPI_RAND_SEED);
+
+       SIMIX_global_init(&argc, argv);
+       SIMIX_function_register("smpi_simulated_main", smpi_simulated_main);
+       SIMIX_create_environment(argv[1]);
+       SIMIX_launch_application(argv[2]);
+
+       /* Prepare to display some more info when dying on Ctrl-C pressing */
+       //signal(SIGINT, inthandler);
+
+       /* Clean IO before the run */
+       fflush(stdout);
+       fflush(stderr);
+
+       while (SIMIX_solve(actions_done, actions_failed) != -1.0) {
+               while (action = xbt_fifo_pop(actions_failed)) {
+                       DEBUG1("** %s failed **", action->name);
+                       while (cond = xbt_fifo_pop(action->cond_list)) {
+                               SIMIX_cond_broadcast(cond);
+                       }
+                       SIMIX_action_destroy(action);
+               }
+               while (action = xbt_fifo_pop(actions_done)) {
+                       DEBUG1("** %s done **",action->name);
+                       while (cond = xbt_fifo_pop(action->cond_list)) {
+                               SIMIX_cond_broadcast(cond);
+                       }
+                       SIMIX_action_destroy(action);
+               }
+       }
+       xbt_fifo_free(actions_failed);
+       xbt_fifo_free(actions_done);
+       INFO1("simulation time %g", SIMIX_get_clock());
+       SIMIX_clean();
+       return 0;
 }
 
-void smpi_isend(smpi_mpi_request_t *sendreq) {
-  int rank = smpi_host_rank_self();
-  if (NULL == smpi_last_pending_send_requests[rank]) {
-    smpi_pending_send_requests[rank] = sendreq;
-  } else {
-    smpi_last_pending_send_requests[rank]->next = sendreq;
-  }
-  smpi_last_pending_send_requests[rank] = sendreq;
-  if (SIMIX_process_is_suspended(smpi_sender_processes[rank])) {
-    SIMIX_process_resume(smpi_sender_processes[rank]);
-  }
+void smpi_mpi_land_func(void *x, void *y, void *z)
+{
+       *(int *)z = *(int *)x && *(int *)y;
 }
 
-void smpi_match_requests(int rank) {
-  smpi_mpi_request_t *frequest, *prequest, *crequest;
-  smpi_received_t *freceived, *preceived, *creceived;
-  size_t dsize;
-  short int match;
-  frequest  = smpi_pending_recv_requests[rank];
-  prequest  = NULL;
-  crequest  = frequest;
-  while(NULL != crequest) {
-    freceived = smpi_received[rank];
-    preceived = NULL;
-    creceived = freceived;
-    match     = 0;
-    while(NULL != creceived && !match) {
-      if(crequest->comm->id == creceived->commid && 
-        (MPI_ANY_SOURCE == crequest->src || crequest->src == creceived->src) && 
-        crequest->tag == creceived->tag) {
-
-        // we have a match!
-        match = 1;
-
-        // pull the request from the queue
-        if(NULL == prequest) {
-          frequest = crequest->next;
-          smpi_pending_recv_requests[rank] = frequest;
-        } else {
-          prequest->next = crequest->next;
-        }
-        if(crequest == smpi_last_pending_recv_requests[rank]) {
-          smpi_last_pending_recv_requests[rank] = prequest;
-        }
-
-        // pull the received data from the queue
-        if(NULL == preceived) {
-          freceived = creceived->next;
-          smpi_received[rank] = freceived;
-        } else {
-          preceived->next = creceived->next;
-        }
-        if(creceived == smpi_last_received[rank]) {
-          smpi_last_received[rank] = preceived;
-        }
-
-        // for when request->src is any source
-        crequest->src = creceived->src;
-
-        // calculate data size
-        dsize = crequest->count * crequest->datatype->size;
-
-        // copy data to buffer
-        memcpy(crequest->buf, creceived->data, dsize);
-
-        // fwd through
-        crequest->fwdthrough = creceived->fwdthrough;
-
-        // get rid of received data node, no longer needed
-        xbt_free(creceived->data);
-        xbt_free(creceived);
-
-        if (crequest->fwdthrough == rank) {
-          smpi_complete(crequest);
-        } else {
-          crequest->src = rank;
-          crequest->dst = (rank + 1) % crequest->comm->size;
-          smpi_isend(crequest);
-        }
-
-      } else {
-        preceived = creceived;
-        creceived = creceived->next;
-      }
-    }
-    prequest = crequest;
-    crequest = crequest->next;
-  }
+void smpi_mpi_sum_func(void *x, void *y, void *z)
+{
+       *(int *)z = *(int *)x + *(int *)y;
 }
 
-void smpi_bench_begin() {
-  xbt_assert0(!smpi_benchmarking, "Already benchmarking");
-  smpi_benchmarking = 1;
-  xbt_os_timer_start(smpi_timer);
-  return;
-}
+int smpi_mpi_rank(smpi_mpi_communicator_t *comm, smx_host_t host)
+{
+       int i;
 
-void smpi_bench_end() {
-  m_task_t ctask = NULL;
-  double duration;
-  xbt_assert0(smpi_benchmarking, "Not benchmarking yet");
-  smpi_benchmarking = 0;
-  xbt_os_timer_stop(smpi_timer);
-  duration = xbt_os_timer_elapsed(smpi_timer);
-  ctask = MSG_task_create("computation", duration * smpi_reference, 0 , NULL);
-  MSG_task_execute(ctask);
-  MSG_task_destroy(ctask);
-  return;
-}
+       for(i = comm->size - 1; i > 0 && host != comm->hosts[i]; i--);
 
-int smpi_create_request(void *buf, int count, smpi_mpi_datatype_t *datatype,
-  int src, int dst, int tag, smpi_mpi_communicator_t *comm, smpi_mpi_request_t **request) {
-  int retval = MPI_SUCCESS;
-  *request = NULL;
-  if (NULL == buf && 0 < count) {
-    retval = MPI_ERR_INTERN;
-  } else if (0 > count) {
-    retval = MPI_ERR_COUNT;
-  } else if (NULL == datatype) {
-    retval = MPI_ERR_TYPE;
-  } else if (NULL == comm) {
-    retval = MPI_ERR_COMM;
-  } else if (MPI_ANY_SOURCE != src && (0 > src || comm->size <= src)) {
-    retval = MPI_ERR_RANK;
-  } else if (0 > dst || comm->size <= dst) {
-    retval = MPI_ERR_RANK;
-  } else if (0 > tag) {
-    retval = MPI_ERR_TAG;
-  } else {
-    *request = xbt_malloc(sizeof(smpi_mpi_request_t));
-    (*request)->buf        = buf;
-    (*request)->count      = count;
-    (*request)->datatype   = datatype;
-    (*request)->src        = src;
-    (*request)->dst        = dst;
-    (*request)->tag        = tag;
-    (*request)->comm       = comm;
-    (*request)->completed  = 0;
-    (*request)->fwdthrough = dst;
-    (*request)->waitlist   = NULL;
-    (*request)->next       = NULL;
-  }
-  return retval;
+       return i;
 }
 
-void smpi_barrier(smpi_mpi_communicator_t *comm) {
-  int i;
-  comm->barrier++;
-  if(comm->barrier < comm->size) {
-    SIMIX_process_suspend(SIMIX_process_self());
-  } else {
-    comm->barrier = 0;
-    for(i = 0; i < comm->size; i++) {
-      if (SIMIX_process_is_suspended(comm->processes[i])) {
-        SIMIX_process_resume(comm->processes[i]);
-      }
-    }
-  }
+int inline smpi_mpi_rank_self(smpi_mpi_communicator_t *comm)
+{
+       return smpi_mpi_rank(comm, SIMIX_host_self());
 }
 
-int smpi_comm_rank(smpi_mpi_communicator_t *comm, smx_host_t host) {
-  int i;
-  for(i = 0; i < comm->size && host != comm->hosts[i]; i++);
-  if (i >= comm->size) i = -1;
-  return i;
+void smpi_mpi_init()
+{
+       int i;
+       int size;
+       smx_host_t *hosts;
+       smx_host_t host;
+       double duration;
+
+       // FIXME: mutex?
+       smpi_running_hosts++;
+
+       // initialize some local variables
+       host  = SIMIX_host_self();
+       hosts = SIMIX_host_get_table();
+       size  = SIMIX_host_get_number();
+
+       // node 0 sets the globals
+       if (host == hosts[0]) {
+
+               // global communicator
+               smpi_mpi_comm_world.size         = size;
+               smpi_mpi_comm_world.barrier      = 0;
+               smpi_mpi_comm_world.hosts        = hosts;
+               smpi_mpi_comm_world.processes    = xbt_new0(smx_process_t, size);
+               smpi_mpi_comm_world.processes[0] = SIMIX_process_self();
+
+               // mpi datatypes
+               smpi_mpi_byte.size               = (size_t)1;
+               smpi_mpi_int.size                = sizeof(int);
+               smpi_mpi_double.size             = sizeof(double);
+
+               // mpi operations
+               smpi_mpi_land.func               = &smpi_mpi_land_func;
+               smpi_mpi_sum.func                = &smpi_mpi_sum_func;
+
+               // smpi globals
+               smpi_pending_send_requests       = xbt_new0(xbt_fifo_t, size);
+               smpi_pending_recv_requests       = xbt_new0(xbt_fifo_t, size);
+               smpi_received_messages           = xbt_new0(xbt_fifo_t, size);
+
+               for(i = 0; i < size; i++) {
+                       smpi_pending_send_requests[i] = xbt_fifo_new();
+                       smpi_pending_recv_requests[i] = xbt_fifo_new();
+                       smpi_received_messages[i]     = xbt_fifo_new();
+               }
+
+               smpi_timer                      = xbt_os_timer_new();
+               smpi_reference_speed            = SMPI_DEFAULT_SPEED;
+               smpi_benchmarking               = 0;
+
+               // FIXME: tell other nodes to initialize, and wait for all clear
+
+               // FIXME: send everyone okay to begin
+
+       } else {
+               // FIMXE: wait for node 0 
+               smpi_mpi_comm_world.processes[smpi_mpi_rank_self(&smpi_mpi_comm_world)] = SIMIX_process_self();
+               // FIXME: signal node 0
+               // FIXME: wait for node 0
+       }
 }
 
-void smpi_irecv(smpi_mpi_request_t *recvreq) {
-  int rank = smpi_host_rank_self();
-  if (NULL == smpi_pending_recv_requests[rank]) {
-    smpi_pending_recv_requests[rank] = recvreq;
-  } else if (NULL != smpi_last_pending_recv_requests[rank]) {
-    smpi_last_pending_recv_requests[rank]->next = recvreq;
-  } else { // can't happen!
-    fprintf(stderr, "smpi_pending_recv_requests not null while smpi_last_pending_recv_requests null!\n");
-  }
-  smpi_last_pending_recv_requests[rank] = recvreq;
-  smpi_match_requests(rank);
-  if (SIMIX_process_is_suspended(smpi_receiver_processes[rank])) {
-    SIMIX_process_resume(smpi_receiver_processes[rank]);
-  }
+void smpi_mpi_finalize()
+{
+       int i;
+
+       // FIXME: mutex?
+       smpi_running_hosts--;
+
+       if (0 >= smpi_running_hosts) {
+               for (i = 0 ; i < smpi_mpi_comm_world.size; i++) {
+                       xbt_fifo_free(smpi_pending_send_requests[i]);
+                       xbt_fifo_free(smpi_pending_recv_requests[i]);
+                       xbt_fifo_free(smpi_received_messages[i]);
+               }
+               xbt_free(smpi_pending_send_requests);
+               xbt_free(smpi_pending_recv_requests);
+               xbt_free(smpi_received_messages);
+               xbt_free(smpi_mpi_comm_world.processes);
+               xbt_os_timer_free(smpi_timer);
+       }
 }
 
-void smpi_wait(smpi_mpi_request_t *request, smpi_mpi_status_t *status) {
-  smpi_waitlist_node_t *waitnode, *current;
-  if (NULL != request) {
-    if (!request->completed) {
-      waitnode = xbt_malloc(sizeof(smpi_waitlist_node_t));
-      waitnode->process = SIMIX_process_self();
-      waitnode->next    = NULL;
-      if (NULL == request->waitlist) {
-        request->waitlist = waitnode;
-      } else {
-        for(current = request->waitlist; NULL != current->next; current = current->next);
-        current->next = waitnode;
-      }
-      SIMIX_process_suspend(waitnode->process);
-    }
-    if (NULL != status && MPI_STATUS_IGNORE != status) {
-      status->MPI_SOURCE = request->src;
-    }
-  }
+void smpi_bench_begin()
+{
+       xbt_assert0(!smpi_benchmarking, "Already benchmarking");
+       smpi_benchmarking = 1;
+       xbt_os_timer_start(smpi_timer);
+       return;
 }
 
-void smpi_wait_all(int count, smpi_mpi_request_t **requests, smpi_mpi_status_t *statuses) {
-  int i;
-  for (i = 0; i < count; i++) {
-    smpi_wait(requests[i], &statuses[i]);
-  }
+void smpi_bench_end()
+{
+       double duration;
+       xbt_assert0(smpi_benchmarking, "Not benchmarking yet");
+       smpi_benchmarking = 0;
+       xbt_os_timer_stop(smpi_timer);
+       duration = xbt_os_timer_elapsed(smpi_timer);
+       // FIXME: add simix call to perform computation
+       return;
 }
 
-void smpi_wait_all_nostatus(int count, smpi_mpi_request_t **requests) {
-  int i;
-  for (i = 0; i < count; i++) {
-    smpi_wait(requests[i], MPI_STATUS_IGNORE);
-  }
-}
-
-int smpi_sender(int argc, char *argv[]) {
-  smx_process_t process;
-  char taskname[50];
-  size_t dsize;
-  void *data;
-  smx_host_t dhost;
-  m_task_t mtask;
-  int rank, fc, ft;
-  smpi_mpi_request_t *sendreq;
-
-  process = SIMIX_process_self();
-
-  // wait for init
-  mtask = (m_task_t)0;
-  MSG_task_get(&mtask, SEND_SYNC_PORT);
-
-  rank = smpi_host_rank_self();
-
-  smpi_sender_processes[rank] = process;
-
-  // ready!
-  MSG_task_put(mtask, MSG_task_get_source(mtask), SEND_SYNC_PORT);
-
-  while (0 < smpi_running_hosts) {
-    sendreq = smpi_pending_send_requests[rank];
-    if (NULL != sendreq) {
-
-      // pull from queue if not a fwd or no more to fwd
-      if (sendreq->dst == sendreq->fwdthrough) {
-        smpi_pending_send_requests[rank] = sendreq->next;
-        if(sendreq == smpi_last_pending_send_requests[rank]) {
-          smpi_last_pending_send_requests[rank] = NULL;
-        }
-        ft = sendreq->dst;
-      } else {
-        fc = ((sendreq->fwdthrough - sendreq->dst + sendreq->comm->size) % sendreq->comm->size) / 2;
-        ft = (sendreq->dst + fc) % sendreq->comm->size;
-        //printf("node %d sending broadcast to node %d through node %d\n", rank, sendreq->dst, ft);
-      }
-
-      // create task to send
-      sprintf(taskname, "comm:%d,src:%d,dst:%d,tag:%d,ft:%d", sendreq->comm->id, sendreq->src, sendreq->dst, sendreq->tag, ft);
-      dsize = sendreq->count * sendreq->datatype->size;
-      data  = xbt_malloc(dsize);
-      memcpy(data, sendreq->buf, dsize);
-      mtask = MSG_task_create(taskname, 0, dsize, data);
-
-      // figure out which host to send it to
-      dhost = sendreq->comm->hosts[sendreq->dst];
-
-      // send task
-      #ifdef DEBUG
-        printf("host %s attempting to send to host %s\n", SIMIX_host_get_name(SIMIX_host_self()), SIMIX_host_get_name(dhost));
-      #endif
-      MSG_task_put(mtask, dhost, MPI_PORT);
-
-      if (sendreq->dst == sendreq->fwdthrough) {
-        smpi_complete(sendreq);
-      } else {
-        sendreq->dst = (sendreq->dst + fc + 1) % sendreq->comm->size;
-      }
-
-    } else {
-      SIMIX_process_suspend(process);
-    }
-  }
-  return 0;
+void smpi_barrier(smpi_mpi_communicator_t *comm) {
+       int i;
+       // FIXME: mutex
+       comm->barrier++;
+       if(comm->barrier < comm->size) {
+               SIMIX_process_suspend(SIMIX_process_self());
+       } else {
+               comm->barrier = 0;
+               for(i = 0; i < comm->size; i++) {
+                       if (SIMIX_process_is_suspended(comm->processes[i])) {
+                               SIMIX_process_resume(comm->processes[i]);
+                       }
+               }
+       }
 }
 
-int smpi_receiver(int argc, char **argv) {
-  smx_process_t process;
-  m_task_t mtask;
-  smpi_received_t *received;
-  int rank;
-  smpi_mpi_request_t *recvreq;
-
-  process = SIMIX_process_self();
-
-  // wait for init
-  mtask = (m_task_t)0;
-  MSG_task_get(&mtask, RECV_SYNC_PORT);
-
-  rank = smpi_host_rank_self();
-
-  // potential race condition...
-  smpi_receiver_processes[rank] = process;
-
-  // ready!
-  MSG_task_put(mtask, MSG_task_get_source(mtask), RECV_SYNC_PORT);
-
-  while (0 < smpi_running_hosts) {
-    recvreq = smpi_pending_recv_requests[rank];
-    if (NULL != recvreq) {
-      mtask = (m_task_t)0;
-
-      #ifdef DEBUG
-        printf("host %s waiting to receive from anyone, but first in queue is (%d,%d,%d).\n",
-         SIMIX_host_get_name(SIMIX_host_self()), recvreq->src, recvreq->dst, recvreq->tag);
-      #endif
-      MSG_task_get(&mtask, MPI_PORT);
-
-      received = xbt_malloc(sizeof(smpi_received_t));
-
-      sscanf(MSG_task_get_name(mtask), "comm:%d,src:%d,dst:%d,tag:%d,ft:%d",
-        &received->commid, &received->src, &received->dst, &received->tag, &received->fwdthrough);
-      received->data = MSG_task_get_data(mtask);
-      received->next = NULL;
-
-      if (NULL == smpi_last_received[rank]) {
-        smpi_received[rank] = received;
-      } else {
-        smpi_last_received[rank]->next = received;
-      }
-      smpi_last_received[rank] = received;
-
-      MSG_task_destroy(mtask);
-
-      smpi_match_requests(rank);
-
-    } else {
-      SIMIX_process_suspend(process);
-    }
-  }
-  return 0;
+int smpi_comm_rank(smpi_mpi_communicator_t *comm, smx_host_t host)
+{
+       int i;
+       for(i = 0; i < comm->size && host != comm->hosts[i]; i++);
+       if (i >= comm->size) i = -1;
+       return i;
 }
 
 // FIXME: move into own file
-int smpi_gettimeofday(struct timeval *tv, struct timezone *tz) {
-  double now;
-  int retval = 0;
-  smpi_bench_end();
-  if (NULL == tv) {
-    retval = -1;
-  } else {
-    now = SIMIX_get_clock();
-    tv->tv_sec  = now;
-    tv->tv_usec = ((now - (double)tv->tv_sec) * 1000000.0);
-  }
-  smpi_bench_begin();
-  return retval;
+int smpi_gettimeofday(struct timeval *tv, struct timezone *tz)
+{
+       double now;
+       int retval = 0;
+       smpi_bench_end();
+       if (NULL == tv) {
+               retval = -1;
+       } else {
+               now = SIMIX_get_clock();
+               tv->tv_sec  = now;
+               tv->tv_usec = ((now - (double)tv->tv_sec) * 1000000.0);
+       }
+       smpi_bench_begin();
+       return retval;
 }
 
-unsigned int smpi_sleep(unsigned int seconds) {
-  m_task_t task = NULL;
-  smpi_bench_end();
-  task = MSG_task_create("sleep", seconds * DEFAULT_POWER, 0, NULL);
-  MSG_task_execute(task);
-  MSG_task_destroy(task);
-  smpi_bench_begin();
-  return 0;
+unsigned int smpi_sleep(unsigned int seconds)
+{
+       smx_host_t self;
+       smx_action_t sleep_action;
+
+       smpi_bench_end();
+       // FIXME: simix sleep
+       self = SIMIX_host_self();
+       sleep_action = SIMIX_action_sleep(self, seconds);
+       smpi_bench_begin();
+       return 0;
 }
 
-void smpi_exit(int status) {
-  smpi_bench_end();
-  smpi_running_hosts--;
-  SIMIX_process_kill(SIMIX_process_self());
-  return;
+void smpi_exit(int status)
+{
+       smpi_bench_end();
+       // FIXME: mutex
+       smpi_running_hosts--;
+       SIMIX_process_kill(SIMIX_process_self());
+       return;
 }
index 924bbde..e5af057 100644 (file)
 #include <stdio.h>
 #include <sys/time.h>
-#include "msg/msg.h"
-#include "xbt/sysdep.h"
-#include "xbt/xbt_portability.h"
 #include "smpi.h"
 
-int MPI_Init(int *argc, char ***argv) {
-  smpi_mpi_init();
-  smpi_bench_begin();
-  return MPI_SUCCESS;
+int MPI_Init(int *argc, char ***argv)
+{
+       smpi_mpi_init();
+       smpi_bench_begin();
+       return MPI_SUCCESS;
 }
 
-int MPI_Finalize() {
-  smpi_bench_end();
-  smpi_mpi_finalize();
-  return MPI_SUCCESS;
+int MPI_Finalize()
+{
+       smpi_bench_end();
+       smpi_mpi_finalize();
+       return MPI_SUCCESS;
 }
 
 // right now this just exits the current node, should send abort signal to all
 // hosts in the communicator;
-int MPI_Abort(MPI_Comm comm, int errorcode) {
-  smpi_exit(errorcode);
+int MPI_Abort(MPI_Comm comm, int errorcode)
+{
+       smpi_exit(errorcode);
 }
 
-int MPI_Comm_size(MPI_Comm comm, int *size) {
-  int retval = MPI_SUCCESS;
-  smpi_bench_end();
-  if (NULL == comm) {
-    retval = MPI_ERR_COMM;
-  } else if (NULL == size) {
-    retval = MPI_ERR_ARG;
-  } else {
-    *size = comm->size;
-  }
-  smpi_bench_begin();
-  return retval;
-}
-
-int MPI_Comm_rank(MPI_Comm comm, int *rank) {
-  int retval = MPI_SUCCESS;
-  smpi_bench_end();
-  if (NULL == comm) {
-    retval = MPI_ERR_COMM;
-  } else if (NULL == rank) {
-    retval = MPI_ERR_ARG;
-  } else {
-    *rank = smpi_comm_rank(comm, SIMIX_host_self());
-  }
-  smpi_bench_begin();
-  return retval;
-}
-
-/*
-int MPI_Comm_split(MPI_Comm comm, int color, int key, MPI_Comm *newcomm) {
-  int retval = MPI_SUCCESS;
-  m_host_t host = SIMIX_host_self();
-  int rank = smpi_comm_rank(comm, host);
-  smpi_mpi_comm_split_table_node_t *split_table; 
-  split_table = xbt_malloc(sizeof(smpi_mpi_comm_split_table_node_t) * comm->size);
-  split_table[rank].color = color;
-  split_table[rank].key   = key;
-  split_table[rank].host  = host;
-  smpi_mpi_alltoall(
-  return retval;
-}
-*/
-
-int MPI_Type_size(MPI_Datatype datatype, size_t *size) {
-  int retval = MPI_SUCCESS;
-  smpi_bench_end();
-  if (NULL == datatype) {
-    retval = MPI_ERR_TYPE;
-  } else if (NULL == size) {
-    retval = MPI_ERR_ARG;
-  } else {
-    *size = datatype->size;
-  }
-  smpi_bench_begin();
-  return retval;
-}
-
-int MPI_Wait(MPI_Request *request, MPI_Status *status) {
-  int retval = MPI_SUCCESS;
-  smpi_bench_end();
-  if (NULL == request) {
-    retval = MPI_ERR_REQUEST;
-  } else if (NULL == status) {
-    retval = MPI_ERR_ARG;
-  } else {
-    smpi_wait(*request, status);
-  }
-  smpi_bench_begin();
-  return retval;
-}
-
-int MPI_Waitall(int count, MPI_Request *requests, MPI_Status *statuses) {
-  int retval = MPI_SUCCESS;
-  smpi_bench_end();
-  if (NULL == requests) {
-    retval = MPI_ERR_REQUEST;
-  } else if (NULL == statuses) {
-    retval = MPI_ERR_ARG;
-  } else {
-    smpi_wait_all(count, requests, statuses);
-  }
-  smpi_bench_begin();
-  return MPI_ERR_INTERN;
-}
+int MPI_Comm_size(MPI_Comm comm, int *size)
+{
+       int retval = MPI_SUCCESS;
 
-int MPI_Irecv(void *buf, int count, MPI_Datatype datatype, int src, int tag, MPI_Comm comm, MPI_Request *request) {
-  int retval = MPI_SUCCESS;
-  int dst;
-  smpi_mpi_request_t *recvreq;
-  smpi_bench_end();
-  dst = smpi_comm_rank(comm, SIMIX_host_self());
-  retval = smpi_create_request(buf, count, datatype, src, dst, tag, comm, &recvreq);
-  if (NULL != recvreq) {
-    smpi_irecv(recvreq);
-    if (NULL != request) {
-      *request = recvreq;
-    }
-  }
-  smpi_bench_begin();
-  return retval;
-}
+       smpi_bench_end();
 
-int MPI_Recv(void *buf, int count, MPI_Datatype datatype, int src, int tag, MPI_Comm comm, MPI_Status *status) {
-  int retval = MPI_SUCCESS;
-  int dst;
-  smpi_mpi_request_t *recvreq;
-  smpi_bench_end();
-  dst = smpi_comm_rank(comm, SIMIX_host_self());
-  retval = smpi_create_request(buf, count, datatype, src, dst, tag, comm, &recvreq);
-  if (NULL != recvreq) {
-    smpi_irecv(recvreq);
-    smpi_wait(recvreq, status);
-    xbt_free(recvreq);
-  }
-  smpi_bench_begin();
-  return retval;
-}
+       if (NULL == comm) {
+               retval = MPI_ERR_COMM;
+       } else if (NULL == size) {
+               retval = MPI_ERR_ARG;
+       } else {
+               *size = comm->size;
+       }
 
-int MPI_Isend(void *buf, int count, MPI_Datatype datatype, int dst, int tag, MPI_Comm comm, MPI_Request *request) {
-  int retval = MPI_SUCCESS;
-  int src;
-  smpi_mpi_request_t *sendreq;
-  smpi_bench_end();
-  src = smpi_comm_rank(comm, SIMIX_host_self());
-  retval = smpi_create_request(buf, count, datatype, src, dst, tag, comm, &sendreq);
-  if (NULL != sendreq) {
-    smpi_isend(sendreq);
-    if (NULL != request) {
-      *request = sendreq;
-    }
-  }
-  smpi_bench_begin();
-  return retval;
-}
+       smpi_bench_begin();
 
-int MPI_Send(void *buf, int count, MPI_Datatype datatype, int dst, int tag, MPI_Comm comm) {
-  int retval = MPI_SUCCESS;
-  int src;
-  smpi_mpi_request_t *sendreq;
-  smpi_bench_end();
-  src = smpi_comm_rank(comm, SIMIX_host_self());
-  retval = smpi_create_request(buf, count, datatype, src, dst, tag, comm, &sendreq);
-  if (NULL != sendreq) {
-    smpi_isend(sendreq);
-    smpi_wait(sendreq, MPI_STATUS_IGNORE);
-  }
-  smpi_bench_begin();
-  return retval;
+       return retval;
 }
 
-int MPI_Bcast(void *buf, int count, MPI_Datatype datatype, int root, MPI_Comm comm) {
-  int i;
-  int rank;
-  smpi_mpi_request_t *request;
-  smpi_bench_end();
+int MPI_Comm_rank(MPI_Comm comm, int *rank)
+{
+       int retval = MPI_SUCCESS;
 
-  rank = smpi_comm_rank(comm, SIMIX_host_self());
+       smpi_bench_end();
 
-  if (root == rank) {
-    smpi_create_request(buf, count, datatype, root, (rank + 1) % comm->size, 0, comm, &request);
-    if (comm->size > 2) {
-      request->fwdthrough = (rank - 1 + comm->size) % comm->size;
-    }
-    smpi_isend(request);
-    smpi_wait(request, MPI_STATUS_IGNORE);
-  } else {
-    smpi_create_request(buf, count, datatype, MPI_ANY_SOURCE, rank, 0, comm, &request);
-    if (NULL != request) {
-      smpi_irecv(request);
-      smpi_wait(request, MPI_STATUS_IGNORE);
-    }
-  }
+       if (NULL == comm) {
+               retval = MPI_ERR_COMM;
+       } else if (NULL == rank) {
+               retval = MPI_ERR_ARG;
+       } else {
+               *rank = smpi_comm_rank(comm, SIMIX_host_self());
+       }
 
-  smpi_bench_begin();
-  return MPI_SUCCESS;
-}
+       smpi_bench_begin();
 
-int MPI_Barrier(MPI_Comm comm) {
-  smpi_bench_end();
-  smpi_barrier(comm);
-  smpi_bench_begin();
-  return MPI_SUCCESS;
+       return retval;
 }
 
-// FIXME: instead of everyone sending in order, might be a good idea to send to next...
-int MPI_Alltoall(void *sendbuf, int sendcount, MPI_Datatype sendtype, void *recvbuf, int recvcount, MPI_Datatype recvtype, MPI_Comm comm) {
-  int i;
-  int rank;
-  smpi_mpi_request_t **sendreqs, **recvreqs;
-  smpi_bench_end();
-
-  rank = smpi_comm_rank(comm, SIMIX_host_self());
-
-  sendreqs = xbt_malloc(sizeof(smpi_mpi_request_t*) * comm->size);
-  recvreqs = xbt_malloc(sizeof(smpi_mpi_request_t*) * comm->size);
+int MPI_Type_size(MPI_Datatype datatype, size_t *size)
+{
+       int retval = MPI_SUCCESS;
 
-  for (i = 0; i < comm->size; i++) {
-    if (rank == i) {
-      sendreqs[i] = NULL;
-      recvreqs[i] = NULL;
-      memcpy(recvbuf + recvtype->size * recvcount * i, sendbuf + sendtype->size * sendcount * i, recvtype->size * recvcount);
-    } else {
-      smpi_create_request(sendbuf + sendtype->size * sendcount * i, sendcount, sendtype, rank, i, 0, comm, sendreqs + i);
-      smpi_isend(sendreqs[i]);
-      smpi_create_request(recvbuf + recvtype->size * recvcount * i, recvcount, recvtype, i, rank, 0, comm, recvreqs + i);
-      smpi_irecv(recvreqs[i]);
-    }
-  } 
+       smpi_bench_end();
 
-  smpi_wait_all_nostatus(comm->size, sendreqs);
-  smpi_wait_all_nostatus(comm->size, recvreqs);
+       if (NULL == datatype) {
+               retval = MPI_ERR_TYPE;
+       } else if (NULL == size) {
+               retval = MPI_ERR_ARG;
+       } else {
+               *size = datatype->size;
+       }
 
-  xbt_free(sendreqs);
-  xbt_free(recvreqs);
+       smpi_bench_begin();
 
-  smpi_bench_begin();
-  return MPI_SUCCESS;
+       return retval;
 }
 
-// FIXME: mpi routines shouldn't call mpi routines, complexity belongs at a lower level
-// also, there's probably a really clever way to overlap everything for this one...
-int MPI_Allreduce(void *sendbuf, void *recvbuf, int count, MPI_Datatype datatype, MPI_Op op, MPI_Comm comm) {
-  MPI_Reduce(sendbuf, recvbuf, count, datatype, op, 0, comm);
-  MPI_Bcast(recvbuf, count, datatype, 0, comm);
-  return MPI_SUCCESS;
-}
-
-// FIXME: check if behavior defined when send/recv bufs are same...
-int MPI_Reduce(void *sendbuf, void *recvbuf, int count, MPI_Datatype datatype, MPI_Op op, int root, MPI_Comm comm) {
-  int i, j;
-  int rank;
-  smpi_mpi_request_t **requests;
-  void **scratchbuf;
-  smpi_bench_end();
-
-  rank = smpi_comm_rank(comm, SIMIX_host_self());
-
-  if (root == rank) {
-    requests = xbt_malloc(sizeof(smpi_mpi_request_t*) * comm->size);
-    scratchbuf = xbt_malloc(sizeof(void*) * comm->size);
-    memcpy(recvbuf, sendbuf, datatype->size * count);
-    for (i = 0; i < comm->size; i++) {
-      if (rank == i) {
-        requests[i] = NULL;
-        scratchbuf[i] = NULL;
-      } else {
-        scratchbuf[i] = xbt_malloc(datatype->size * count);
-        smpi_create_request(scratchbuf[i], count, datatype, MPI_ANY_SOURCE, rank, 0, comm, requests + i);
-        smpi_irecv(requests[i]);
-      }
-    }
-    smpi_wait_all_nostatus(comm->size, requests); // FIXME: use wait_any for slight performance gain
-    for (i = 0; i < comm->size; i++) {
-      if (rank != i) {
-        for (j = 0; j < count; j++) {
-          op->func(scratchbuf[i] + datatype->size * j, recvbuf + datatype->size * j, recvbuf + datatype->size * j);
-        }
-        xbt_free(requests[i]);
-        xbt_free(scratchbuf[i]);
-      }
-    }
-    xbt_free(requests);
-    xbt_free(scratchbuf);
-  } else {
-    requests = xbt_malloc(sizeof(smpi_mpi_request_t*));
-    smpi_create_request(sendbuf, count, datatype, rank, root, 0, comm, requests);
-    smpi_isend(*requests);
-    smpi_wait(*requests, MPI_STATUS_IGNORE);
-    xbt_free(*requests);
-    xbt_free(requests);
-  }
-
-  smpi_bench_begin();
-  return MPI_SUCCESS;
+int MPI_Barrier(MPI_Comm comm)
+{
+       smpi_bench_end();
+       smpi_barrier(comm);
+       smpi_bench_begin();
+       return MPI_SUCCESS;
 }
diff --git a/src/smpi/src/smpi_new.c b/src/smpi/src/smpi_new.c
new file mode 100644 (file)
index 0000000..e69de29